----------------------------------------------------------- This is an automatically generated e-mail. To reply, visit: https://reviews.apache.org/r/45324/#review128249 -----------------------------------------------------------
Here's a first round. Some JavaDoc or a reverence to any existing design doc would help the reviewers. Also, the Jira says there are examples in this patch but I didn't see them. Can you point them out? They may be a good starting place. It seems that this patch builds on another patch, so we can't apply this one on master. Please direct us to the other patch. What are the plans for using reactive streams? Is it just to formalize the DAG of operators or is there more to it than that? samza-operator/src/main/java/org/apache/samza/operators/exception/OperatorException.java (line 33) <https://reviews.apache.org/r/45324/#comment191663> A valid UID should be generated samza-operator/src/main/java/org/apache/samza/operators/exception/OperatorException.java (line 52) <https://reviews.apache.org/r/45324/#comment198827> It might be worth renaming these static constructors to distinguish them from getError(). Two options come to mind: 1. createInputDisabledException() and similar for TooOld 2. Considering the user perspective, I think OperatorException.inputDisabled() reads well. samza-operator/src/main/java/org/apache/samza/operators/factory/DataStream.java (line 30) <https://reviews.apache.org/r/45324/#comment191699> Doc missing from this entire class, actually most of the patch. It's definitely needed before submitting, but would make review much easier too. samza-operator/src/main/java/org/apache/samza/operators/scan/Scanner.java (line 24) <https://reviews.apache.org/r/45324/#comment198847> It's unclear to me what a scanner is. I can see that it operates on a SSP, but does it filter the SSP, duplicate it, act as a cursor? samza-operator/src/main/java/org/apache/samza/operators/scan/Scanner.java (line 32) <https://reviews.apache.org/r/45324/#comment198846> Is this method filtering based on a key or is the key a timestamp? It's unclear to me why the method is called timestamp and why it needs to be chainable (I assume that's why it's stubbed to return "this") Same comment for messageKey() below samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Tuple.java (line 38) <https://reviews.apache.org/r/45324/#comment198861> The terminology is a bit mixed here. Tuple<K, T>: K suggests "Key" T suggests "Tuple" But then that's a Tuple containing a Tuple, which is awkward. And then on this line we have T getMessage() implying T is a message. I think these constructs need to be clarified. Perhaps this interface is KeyedTuple<K, T> and the method is T getTuple()? samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Tuple.java (line 59) <https://reviews.apache.org/r/45324/#comment198862> I think it'll be useful to always qualify timestamps as "eventTimeStamp" or "systemTimeStamp" samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/SimpleOperator.java (line 22) <https://reviews.apache.org/r/45324/#comment198864> JoinCondition is spelled out below. For consistency, I think JoinFunction should be spelled out here. samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/SimpleOperator.java (line 26) <https://reviews.apache.org/r/45324/#comment198863> "JoinCondition" no 's' samza-sql-core/src/main/java/org/apache/samza/sql/operators/factory/NoopOperatorCallback.java (line 37) <https://reviews.apache.org/r/45324/#comment198865> I don't have an intuition for what this method does or what a Subscriber is. Does it just register a downstream listener? samza-sql-core/src/main/java/org/apache/samza/sql/operators/window/WindowState.java (line 27) <https://reviews.apache.org/r/45324/#comment198859> Are there going to be other, non-Stream pipelines? If not, I think we can drop "Stream" from the class name samza-sql-core/src/main/java/org/apache/samza/sql/operators/window/WindowState.java (line 35) <https://reviews.apache.org/r/45324/#comment198860> Curious why this class uses a message collector rather than output/chaining samza-sql-core/src/main/java/org/apache/samza/sql/window/storage/OrderedStoreKey.java (line 22) <https://reviews.apache.org/r/45324/#comment198858> Will Batch need to be templated with generics? Or is it really just a window cardinality? samza-sql-core/src/main/java/org/apache/samza/system/sql/Offset.java (line 20) <https://reviews.apache.org/r/45324/#comment198857> Alt package: org.apache.samza.stream - Jake Maes On March 28, 2016, 6:59 p.m., Yi Pan (Data Infrastructure) wrote: > > ----------------------------------------------------------- > This is an automatically generated e-mail. To reply, visit: > https://reviews.apache.org/r/45324/ > ----------------------------------------------------------- > > (Updated March 28, 2016, 6:59 p.m.) > > > Review request for samza, Boris Shkolnik, Chris Pettitt, Chinmay Soman, Jake > Maes, Navina Ramesh, Jagadish Venkatraman, and Xinyu Liu. > > > Bugs: SAMZA-914 > https://issues.apache.org/jira/browse/SAMZA-914 > > > Repository: samza > > > Description > ------- > > SAMZA-914: Initial draft for Java programming APIs on operators supporting > DAGs > > > Diffs > ----- > > build.gradle 16facbbf4dff378c561461786ff186bd9e0000ed > gradle/dependency-versions.gradle 52e25aa53a1edc85d478b48898621b26508ad4bb > > samza-operator/src/main/java/org/apache/samza/operators/exception/OperatorException.java > PRE-CREATION > > samza-operator/src/main/java/org/apache/samza/operators/factory/DataStream.java > PRE-CREATION > samza-operator/src/main/java/org/apache/samza/operators/scan/Scanner.java > PRE-CREATION > > samza-operator/src/main/java/org/apache/samza/operators/window/SessionWindow.java > PRE-CREATION > samza-operator/src/main/java/org/apache/samza/operators/window/Timeout.java > PRE-CREATION > samza-operator/src/main/java/org/apache/samza/operators/window/Window.java > PRE-CREATION > samza-operator/src/main/java/org/apache/samza/task/DataStreamTask.java > PRE-CREATION > samza-operator/src/test/java/org/apache/samza/task/DataStreamJoinTask.java > PRE-CREATION > samza-operator/src/test/java/org/apache/samza/task/DataStreamSplitTask.java > PRE-CREATION > samza-operator/src/test/java/org/apache/samza/task/DataStreamUserTask.java > PRE-CREATION > > samza-sql-calcite/src/test/java/org/apache/samza/sql/calcite/schema/TestAvroSchemaConverter.java > PRE-CREATION > samza-sql-core/README.md PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Data.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/api/data/EntityName.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Relation.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Schema.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Stream.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Table.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/api/data/Tuple.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/Operator.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/OperatorCallback.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/OperatorRouter.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/OperatorSpec.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/SimpleOperator.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/api/operators/SqlOperatorFactory.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/data/IncomingMessageTuple.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/data/avro/AvroData.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/sql/data/avro/AvroSchema.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/data/serializers/SqlAvroSerde.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/data/serializers/SqlAvroSerdeFactory.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/data/serializers/SqlStringSerde.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/data/serializers/SqlStringSerdeFactory.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/data/string/StringData.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/data/string/StringSchema.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/factory/NoopOperatorCallback.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/factory/SimpleOperatorFactoryImpl.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/factory/SimpleOperatorImpl.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/factory/SimpleOperatorSpec.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/factory/SimpleRouter.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/join/StreamStreamJoin.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/join/StreamStreamJoinSpec.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/partition/PartitionOp.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/partition/PartitionSpec.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/window/BoundedTimeWindow.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/window/WindowSpec.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/operators/window/WindowState.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/sql/window/storage/OrderedStoreKey.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/system/sql/LongOffset.java > PRE-CREATION > samza-sql-core/src/main/java/org/apache/samza/system/sql/Offset.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/task/sql/RouterMessageCollector.java > PRE-CREATION > > samza-sql-core/src/main/java/org/apache/samza/task/sql/SimpleMessageCollector.java > PRE-CREATION > > samza-sql-core/src/test/java/org/apache/samza/sql/data/serializers/SqlAvroSerdeTest.java > PRE-CREATION > > samza-sql-core/src/test/java/org/apache/samza/task/sql/RandomWindowOperatorTask.java > PRE-CREATION > samza-sql-core/src/test/java/org/apache/samza/task/sql/StreamSqlTask.java > PRE-CREATION > > samza-sql-core/src/test/java/org/apache/samza/task/sql/UserCallbacksSqlTask.java > PRE-CREATION > settings.gradle 4c1aa107a11d413777e69bc4e48847b811aff7d2 > > Diff: https://reviews.apache.org/r/45324/diff/ > > > Testing > ------- > > Locally build via ./gradlew clean build > > > Thanks, > > Yi Pan (Data Infrastructure) > >