> On Jan. 6, 2015, 12:48 a.m., Chris Riccomini wrote:
> > samza-sql/src/main/java/org/apache/samza/sql/api/data/StreamSpec.java, line 
> > 3
> > <https://reviews.apache.org/r/29592/diff/1/?file=806957#file806957line3>
> >
> >     What's the idea behind this class. It feels like a place holder for 
> > something important, but I'm not quite sure what.
> 
> Yi Pan (Data Infrastructure) wrote:
>     Yes. It is a placeholder. It should specify the in-memory stream of 
> tuples that flow between operators.
> 
> Milinda Pathirage wrote:
>     I hope, you mean we are going to add information related to stream 
> definition (fields and their types in a tuple of this stream) here.

@Milinda, yes. That would be one of the option. Chris and I were also talking 
about some potential use cases where tuples with different schema are published 
and received via a single physical stream. In that case, we may have to add the 
schema info to each tuple. We are still thinking about it.


> On Jan. 6, 2015, 12:48 a.m., Chris Riccomini wrote:
> > samza-sql/src/test/java/org/apache/samza/sql/task/StreamSqlTask.java, line 
> > 169
> > <https://reviews.apache.org/r/29592/diff/1/?file=806989#file806989line169>
> >
> >     It's unclear to me where the input/output are. It seems like things are 
> > wired in reverse (i.e. input goes to join, which goes to the window 
> > operators). I would have expected the input to go to the window operators, 
> > and the join to operato on both windowed relations.
> 
> Yi Pan (Data Infrastructure) wrote:
>     When setting the next operator of the current operator, the next operator 
> object has to be created. Hence, the code actually is doing the following:
>     1) create and initialize all operators
>     2) link the operators via setNextOp() from the input operator to the 
> intermediate and final output operators
>     
>     Removing the setNextOp() actually would be a bit more controversal here:
>     If the constructor of the current operator requires the nextOp as the 
> input parameter, that would require the creation of the operators in the 
> reverse order: i.e. from the final output operator backward to the input 
> operator(s);
>     
>     We could use:
>     1) only requires the OperatorSpec as the nextOpSpec in the constructor of 
> operators to avoid the need to create the operator objects in reverse order
>     2) use init() method to set the nextOp.
>     
>     I will experiment on both options. Any other suggestions?
> 
> Milinda Pathirage wrote:
>     What I understood by looking at this code is whole query is executed 
> inside a single stream task. What the advatages of following this method? Is 
> there any plans to distribute the query execution?

@Milinda, yes. The current implementation is focused on the operator layer and 
each operator is executed inside a single stream task. A much 
larger/distributed stream processing topology is the task of a query planner to 
identify the boundaries of each task via stream outputs in the relation 
algebra, and configure/deploy each task with the operators within each task 
boundaries to a single node. Parallel deployment of the same task with 
different configuration (i.e. the same join task that operates on different 
partition of a stream) is also possible, without the need to have the operators 
in different tasks to coordinate with each other if the stream is partitioned 
based on join key. Hence, for the implementation of operators, we can simply 
focus on a single-node implementation. At this level, we wouldn't need to 
consider implementing an operator that requires distributed coordination, 
unless you have certain use cases in mind that requires a distributed operator.


- Yi


-----------------------------------------------------------
This is an automatically generated e-mail. To reply, visit:
https://reviews.apache.org/r/29592/#review66751
-----------------------------------------------------------


On Jan. 5, 2015, 10:19 p.m., Yi Pan (Data Infrastructure) wrote:
> 
> -----------------------------------------------------------
> This is an automatically generated e-mail. To reply, visit:
> https://reviews.apache.org/r/29592/
> -----------------------------------------------------------
> 
> (Updated Jan. 5, 2015, 10:19 p.m.)
> 
> 
> Review request for samza, Chris Riccomini, Navina Ramesh, and Naveen 
> Somasundaram.
> 
> 
> Bugs: SAMZA-482
>     https://issues.apache.org/jira/browse/SAMZA-482
> 
> 
> Repository: samza
> 
> 
> Description
> -------
> 
> StreamSQL operator API draft 
> - This is the first draft of the StreamSQL operator APIs 
> - org.apache.samza.sql.api.* contains definitions of all interface classes 
> - org.apache.samza.sql.operators.* are skeleton implementation of some 
> example build-in operators 
> - src/test/java contains the example application (a stream-join application) 
> using the above APIs
> 
> 
> Diffs
> -----
> 
>   build.gradle 38383bd9e3f0847d6088a4ea4c1ee6f3dcd1e430 
>   samza-sql/src/main/java/org/apache/samza/sql/api/data/Relation.java 
> PRE-CREATION 
>   samza-sql/src/main/java/org/apache/samza/sql/api/data/RelationSpec.java 
> PRE-CREATION 
>   samza-sql/src/main/java/org/apache/samza/sql/api/data/RelationStore.java 
> PRE-CREATION 
>   samza-sql/src/main/java/org/apache/samza/sql/api/data/StreamSpec.java 
> PRE-CREATION 
>   samza-sql/src/main/java/org/apache/samza/sql/api/data/Tuple.java 
> PRE-CREATION 
>   samza-sql/src/main/java/org/apache/samza/sql/api/operators/Operator.java 
> PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/RelationOperator.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/RelationRelationOperator.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/RelationTupleOperator.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/SqlOperatorFactory.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/TupleOperator.java 
> PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/TupleRelationOperator.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/TupleTupleOperator.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/spec/OperatorSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/spec/RelationOperatorSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/spec/RelationRelationSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/spec/RelationTupleSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/spec/TupleOperatorSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/spec/TupleRelationSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/api/operators/spec/TupleTupleSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/factory/SimpleOperatorFactoryImpl.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/output/SystemStreamOp.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/output/SystemStreamSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/partition/PartitionOp.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/partition/PartitionSpec.java
>  PRE-CREATION 
>   samza-sql/src/main/java/org/apache/samza/sql/operators/relation/Join.java 
> PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/relation/JoinSpec.java 
> PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/stream/InsertStream.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/stream/InsertStreamSpec.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/window/BoundedTimeWindow.java
>  PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/window/WindowSpec.java 
> PRE-CREATION 
>   
> samza-sql/src/main/java/org/apache/samza/sql/operators/window/WindowState.java
>  PRE-CREATION 
>   samza-sql/src/main/java/org/apache/samza/task/SqlTaskContext.java 
> PRE-CREATION 
>   samza-sql/src/test/java/org/apache/samza/sql/data/IncomingMessageTuple.java 
> PRE-CREATION 
>   samza-sql/src/test/java/org/apache/samza/sql/store/SQLRelationStore.java 
> PRE-CREATION 
>   samza-sql/src/test/java/org/apache/samza/sql/task/StreamSqlTask.java 
> PRE-CREATION 
>   settings.gradle 3a01fd66359b8c79954ae8f34eeaf4b2e3fdc0b4 
> 
> Diff: https://reviews.apache.org/r/29592/diff/
> 
> 
> Testing
> -------
> 
> 
> Thanks,
> 
> Yi Pan (Data Infrastructure)
> 
>

Reply via email to