Github user sbcd90 commented on a diff in the pull request:
https://github.com/apache/incubator-streams/pull/345#discussion_r94092806
--- Diff:
streams-contrib/streams-persist-cassandra/src/main/java/org/apache/streams/cassandra/CassandraPersistReader.java
---
@@ -0,0 +1,316 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements. See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership. The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ *
+ * http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing,
+ * software distributed under the License is distributed on an
+ * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
+ * KIND, either express or implied. See the License for the
+ * specific language governing permissions and limitations
+ * under the License.
+ */
+
+package org.apache.streams.cassandra;
+
+import org.apache.streams.config.ComponentConfigurator;
+import org.apache.streams.config.StreamsConfigurator;
+import org.apache.streams.core.DatumStatusCounter;
+import org.apache.streams.core.StreamsDatum;
+import org.apache.streams.core.StreamsPersistReader;
+import org.apache.streams.core.StreamsResultSet;
+import org.apache.streams.jackson.StreamsJacksonMapper;
+
+import com.datastax.driver.core.Cluster;
+import com.datastax.driver.core.DataType;
+import com.datastax.driver.core.KeyspaceMetadata;
+import com.datastax.driver.core.Metadata;
+import com.datastax.driver.core.ResultSet;
+import com.datastax.driver.core.Row;
+import com.datastax.driver.core.Session;
+import com.datastax.driver.core.TableMetadata;
+import com.datastax.driver.core.querybuilder.QueryBuilder;
+import com.datastax.driver.core.schemabuilder.SchemaBuilder;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import com.fasterxml.jackson.databind.node.ObjectNode;
+import com.google.common.collect.Queues;
+import org.apache.commons.lang3.StringUtils;
+import org.joda.time.DateTime;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+import java.io.IOException;
+import java.math.BigInteger;
+import java.util.HashMap;
+import java.util.Iterator;
+import java.util.Map;
+import java.util.Objects;
+import java.util.Queue;
+import java.util.concurrent.CompletableFuture;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.ExecutorService;
+import java.util.concurrent.Executors;
+import java.util.concurrent.LinkedBlockingQueue;
+import java.util.concurrent.locks.ReadWriteLock;
+import java.util.concurrent.locks.ReentrantReadWriteLock;
+
+/**
+ * CassandraPersistReader reads documents from cassandra.
+ */
+public class CassandraPersistReader implements StreamsPersistReader {
+
+ public static final String STREAMS_ID = "CassandraPersistReader";
+
+ public static final Logger LOGGER =
LoggerFactory.getLogger(CassandraPersistReader.class);
+
+ protected volatile Queue persistQueue;
+
+ private ObjectMapper mapper = StreamsJacksonMapper.getInstance();
+
+ private ExecutorService executor;
+ private CompletableFuture readerTaskFuture = new
CompletableFuture<>();
+
+ private CassandraConfiguration config;
+
+ protected Cluster cluster;
+ protected Session session;
+
+ protected String keyspace;
+ protected String table;
+ protected Iterator rowIterator;
+
+ protected final ReadWriteLock lock = new ReentrantReadWriteLock();
+
+ /**
+ * CassandraPersistReader constructor - resolves CassandraConfiguration
from JVM 'cassandra'.
+ */
+ public CassandraPersistReader() {
+this.config = new ComponentConfigurator<>(CassandraConfiguration.class)
+
.detectConfiguration(StreamsConfigurator.getConfig().getConfig("cassandra"));
+ }
+
+ /**
+ * CassandraPersistReader constructor - uses supplied
CassandraConfiguration.
+ * @param config config
+ */
+ public CassandraPersistReader(CassandraConfiguration config) {
+this.config = config;
+ }
+
+ /**
+ * CassandraPersistReader constructor - uses supplied persistQueue.
+ * @param persistQueue persistQueue
+ */
+ public CassandraPersistReader(Queue persistQueue) {
+this.config = new ComponentConfigurator<>(CassandraConfiguration.class)
+
.detectConfiguration(StreamsConfigurator.getConfig().getConfig("cassandra"));
+