[ 
https://issues.apache.org/jira/browse/BEAM-6427?focusedWorklogId=185146&page=com.atlassian.jira.plugin.system.issuetabpanels:worklog-tabpanel#worklog-185146
 ]

ASF GitHub Bot logged work on BEAM-6427:
----------------------------------------

                Author: ASF GitHub Bot
            Created on: 15/Jan/19 06:25
            Start Date: 15/Jan/19 06:25
    Worklog Time Spent: 10m 
      Work Description: kennknowles commented on pull request #7504: 
[BEAM-6427] INTERSECT ALL is not compatible with SQL standard.
URL: https://github.com/apache/beam/pull/7504#discussion_r247775196
 
 

 ##########
 File path: 
sdks/java/extensions/sql/src/main/java/org/apache/beam/sdk/extensions/sql/impl/transform/BeamSetOperatorsTransforms.java
 ##########
 @@ -80,8 +80,23 @@ public void processElement(ProcessContext ctx) {
         case INTERSECT:
           if (leftRows.iterator().hasNext() && rightRows.iterator().hasNext()) 
{
             if (all) {
-              for (Row leftRow : leftRows) {
-                ctx.output(leftRow);
+              Iterator<Row> iter = leftRows.iterator();
+              int leftCount = 0;
+              int rightCount = 0;
+              while (iter.hasNext()) {
+                iter.next();
+                leftCount++;
+              }
+              iter = rightRows.iterator();
+              while (iter.hasNext()) {
+                iter.next();
+                rightCount++;
+              }
+
+              // output MIN(m, n)
+              iter = (leftCount <= rightCount) ? leftRows.iterator() : 
rightRows.iterator();
+              while (iter.hasNext()) {
 
 Review comment:
   I had the same thought. To do the `row -> KV<row, count>` without a shuffle 
(and we definitely want to do some version of it before a shuffle) there are 
lots of performance considerations and some nondeterminism. The Dataflow 
implementation is now open source 
[here](https://github.com/apache/beam/blob/master/runners/google-cloud-dataflow-java/worker/src/main/java/org/apache/beam/runners/dataflow/worker/PartialGroupByKeyParDoFns.java)
 and we could do something like that in a `ParDo`. Data consistency requires 
that this be followed up by a full combine. The intermediate results do not 
work as any kind of `Rel` but we _can_ use this as intermediate PCollections 
just for doing set operations. It would have to actually be:
   
   `row -> KV<row, partial count> -> CoGBK -> KV<row, [partial counts], 
[partials counts]>` but small iterables of the partial counts that are easy to 
sum inline with the emit logic.
 
----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
[email protected]


Issue Time Tracking
-------------------

    Worklog Id:     (was: 185146)
    Time Spent: 2h  (was: 1h 50m)

> INTERSECT ALL is not compatible with SQL standard
> -------------------------------------------------
>
>                 Key: BEAM-6427
>                 URL: https://issues.apache.org/jira/browse/BEAM-6427
>             Project: Beam
>          Issue Type: Improvement
>          Components: dsl-sql
>            Reporter: Rui Wang
>            Assignee: Rui Wang
>            Priority: Major
>          Time Spent: 2h
>  Remaining Estimate: 0h
>
> say Row R appears m times on one side and n times on another side. 
> Beam's  INTERSECT ALL is implemented to return MAX(m, n).
> And SQL1999 standard says INTERSECT ALL should return MIN(m, n).



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to