[
https://issues.apache.org/jira/browse/APEXMALHAR-2172?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15413047#comment-15413047
]
ASF GitHub Bot commented on APEXMALHAR-2172:
--------------------------------------------
Github user bhupeshchawda commented on a diff in the pull request:
https://github.com/apache/apex-malhar/pull/358#discussion_r74005624
--- Diff:
library/src/main/java/com/datatorrent/lib/db/jdbc/AbstractJdbcPollInputOperator.java
---
@@ -27,48 +27,62 @@
import java.util.HashMap;
import java.util.List;
import java.util.Map;
-import java.util.concurrent.ArrayBlockingQueue;
+import java.util.concurrent.Executors;
+import java.util.concurrent.LinkedBlockingDeque;
+import java.util.concurrent.ScheduledExecutorService;
+import java.util.concurrent.TimeUnit;
import java.util.concurrent.atomic.AtomicReference;
import javax.validation.constraints.Min;
-
+import javax.validation.constraints.NotNull;
+
+import org.jooq.Condition;
+import org.jooq.DSLContext;
+import org.jooq.Field;
+import org.jooq.SelectField;
+import org.jooq.conf.ParamType;
+import org.jooq.impl.DSL;
+import org.jooq.tools.jdbc.JDBCUtils;
+import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
import org.apache.apex.malhar.lib.wal.FSWindowDataManager;
import org.apache.apex.malhar.lib.wal.WindowDataManager;
import org.apache.commons.lang3.tuple.MutablePair;
import org.apache.hadoop.classification.InterfaceStability.Evolving;
+import com.google.common.annotations.VisibleForTesting;
+
import com.datatorrent.api.Context;
import com.datatorrent.api.Context.OperatorContext;
import com.datatorrent.api.DefaultPartition;
import com.datatorrent.api.Operator.ActivationListener;
-import com.datatorrent.api.Operator.IdleTimeHandler;
import com.datatorrent.api.Partitioner;
+import com.datatorrent.api.annotation.OperatorAnnotation;
import com.datatorrent.api.annotation.Stateless;
import com.datatorrent.lib.db.AbstractStoreInputOperator;
import com.datatorrent.lib.util.KeyValPair;
import com.datatorrent.lib.util.KryoCloneUtils;
import com.datatorrent.netlet.util.DTThrowable;
+import static java.sql.ResultSet.CONCUR_READ_ONLY;
+import static java.sql.ResultSet.TYPE_FORWARD_ONLY;
+import static org.jooq.impl.DSL.field;
+
/**
* Abstract operator for for consuming data using JDBC interface<br>
- * User needs User needs to provide
- * tableName,dbConnection,setEmitColumnList,look-up key <br>
- * Optionally batchSize,pollInterval,Look-up key and a where clause can be
given
- * <br>
- * This operator uses static partitioning to arrive at range queries for
exactly
- * once reads<br>
+ * User needs to provide tableName, dbConnection, columnsExpression,
look-up key<br>
+ * Optionally batchSize, pollInterval and a where clause can be given <br>
+ * This operator uses static partitioning to arrive at range queries for
+ * idempotent reads<br>
* This operator will create a configured number of non-polling static
* partitions for fetching the existing data in the table. And an
additional
* single partition for polling additive data. Assumption is that there is
an
* ordered column using which range queries can be formed<br>
- * If an emitColumnList is provided, please ensure that the keyColumn is
the
+ * If an columnsExpression is provided, please ensure that the keyColumn
is the
* first column in the list<br>
- * Range queries are formed using the {@link JdbcMetaDataUtility}} Output -
- * comma separated list of the emit columns eg columnA,columnB,columnC<br>
- * Only newly added data which has increasing ids will be fetched by the
polling
- * jdbc partition
+ * Only newly added data will be fetched by the polling jdbc partition,
+ * also assumption is rows won't be added or deleted in middle during scan.
*
* In the next iterations this operator would support an in-clause for
--- End diff --
I don't think this can be achieved. Perhaps we should not mention this in
the javadoc.
> Update JDBC poll input operator to fix issues
> ---------------------------------------------
>
> Key: APEXMALHAR-2172
> URL: https://issues.apache.org/jira/browse/APEXMALHAR-2172
> Project: Apache Apex Malhar
> Issue Type: Improvement
> Reporter: Priyanka Gugale
> Assignee: Priyanka Gugale
>
> Update JDBCPollInputOperator to:
> 1. Fix small bugs
> 2. Use jooq query dsl library to construct sql queries
> 3. Make code more readable
> 4. Use row counts rather than key column values to partition reads
--
This message was sent by Atlassian JIRA
(v6.3.4#6332)