You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by mo...@apache.org on 2021/08/25 14:34:14 UTC

[incubator-doris] branch master updated: [BUG] Fixed the materialized number of resultExprs/constExprs and output slot of Union Node is inconsistent (#6380)

This is an automated email from the ASF dual-hosted git repository.

morningman pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-doris.git


The following commit(s) were added to refs/heads/master by this push:
     new 96013de  [BUG] Fixed the materialized number of resultExprs/constExprs and output slot of Union Node is inconsistent (#6380)
96013de is described below

commit 96013decd3800b9864b46cb66d397b2c07579a5f
Author: Xinyi Zou <zo...@foxmail.com>
AuthorDate: Wed Aug 25 22:33:49 2021 +0800

    [BUG] Fixed the materialized number of resultExprs/constExprs and output slot of Union Node is inconsistent (#6380)
---
 .../apache/doris/planner/DistributedPlanner.java   |  1 -
 .../org/apache/doris/planner/SetOperationNode.java | 94 +++++++++++++---------
 .../java/org/apache/doris/planner/PlannerTest.java | 22 +++++
 3 files changed, 80 insertions(+), 37 deletions(-)

diff --git a/fe/fe-core/src/main/java/org/apache/doris/planner/DistributedPlanner.java b/fe/fe-core/src/main/java/org/apache/doris/planner/DistributedPlanner.java
index 7b8efe8..8c3b609 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/planner/DistributedPlanner.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/planner/DistributedPlanner.java
@@ -840,7 +840,6 @@ public class DistributedPlanner {
             childFragment.setOutputPartition(
                     DataPartition.hashPartitioned(setOperationNode.getMaterializedResultExprLists_().get(i)));
         }
-        setOperationNode.init(ctx_.getRootAnalyzer());
         return setOperationFragment;
     }
 
diff --git a/fe/fe-core/src/main/java/org/apache/doris/planner/SetOperationNode.java b/fe/fe-core/src/main/java/org/apache/doris/planner/SetOperationNode.java
index fbd3a48..67ef3b5 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/planner/SetOperationNode.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/planner/SetOperationNode.java
@@ -24,6 +24,7 @@ import org.apache.doris.analysis.SlotRef;
 import org.apache.doris.analysis.TupleDescriptor;
 import org.apache.doris.analysis.TupleId;
 import org.apache.doris.common.CheckedMath;
+import org.apache.doris.common.UserException;
 import org.apache.doris.thrift.TExceptNode;
 import org.apache.doris.thrift.TExplainLevel;
 import org.apache.doris.thrift.TExpr;
@@ -71,7 +72,7 @@ public abstract class SetOperationNode extends PlanNode {
     protected List<List<Expr>> constExprLists_ = Lists.newArrayList();
 
     // Materialized result/const exprs corresponding to materialized slots.
-    // Set in init() and substituted against the corresponding child's output smap.
+    // Set in finalize() and substituted against the corresponding child's output smap.
     protected List<List<Expr>> materializedResultExprLists_ = Lists.newArrayList();
     protected List<List<Expr>> materializedConstExprLists_ = Lists.newArrayList();
 
@@ -126,6 +127,62 @@ public abstract class SetOperationNode extends PlanNode {
     }
 
     @Override
+    public void finalize(Analyzer analyzer) throws UserException {
+        super.finalize(analyzer);
+        // In Doris-6380, moved computePassthrough() and the materialized position of resultExprs/constExprs from this.init()
+        // to this.finalize(), and will not call SetOperationNode::init() again at the end of createSetOperationNodeFragment().
+        //
+        // Reasons for move computePassthrough():
+        //      Because the byteSize of the tuple corresponding to OlapScanNode is updated after
+        //      singleNodePlanner.createSingleNodePlan() and before singleNodePlan.finalize(),
+        //      calling computePassthrough() in SetOperationNode::init() may not be able to accurately determine whether
+        //      the child is pass through. In the previous logic , Will call SetOperationNode::init() again
+        //      at the end of createSetOperationNodeFragment().
+        //
+        // Reasons for move materialized position of resultExprs/constExprs:
+        //      Because the output slot is materialized at various positions in the planner stage, this is to ensure that
+        //      eventually the resultExprs/constExprs and the corresponding output slot have the same materialized state.
+        //      And the order of materialized resultExprs must be the same as the order of child adjusted by
+        //      computePassthrough(), so resultExprs materialized must be placed after computePassthrough().
+
+        // except Node must not reorder the child
+        if (!(this instanceof ExceptNode)) {
+            computePassthrough(analyzer);
+        }
+        // drop resultExprs/constExprs that aren't getting materialized (= where the
+        // corresponding output slot isn't being materialized)
+        materializedResultExprLists_.clear();
+        Preconditions.checkState(resultExprLists_.size() == children.size());
+        List<SlotDescriptor> slots = analyzer.getDescTbl().getTupleDesc(tupleId_).getSlots();
+        for (int i = 0; i < resultExprLists_.size(); ++i) {
+            List<Expr> exprList = resultExprLists_.get(i);
+            List<Expr> newExprList = Lists.newArrayList();
+            Preconditions.checkState(exprList.size() == slots.size());
+            for (int j = 0; j < exprList.size(); ++j) {
+                if (slots.get(j).isMaterialized()) {
+                    newExprList.add(exprList.get(j));
+                }
+            }
+            materializedResultExprLists_.add(
+                    Expr.substituteList(newExprList, getChild(i).getOutputSmap(), analyzer, true));
+        }
+        Preconditions.checkState(
+                materializedResultExprLists_.size() == getChildren().size());
+
+        materializedConstExprLists_.clear();
+        for (List<Expr> exprList : constExprLists_) {
+            Preconditions.checkState(exprList.size() == slots.size());
+            List<Expr> newExprList = Lists.newArrayList();
+            for (int i = 0; i < exprList.size(); ++i) {
+                if (slots.get(i).isMaterialized()) {
+                    newExprList.add(exprList.get(i));
+                }
+            }
+            materializedConstExprLists_.add(newExprList);
+        }
+    }
+
+    @Override
     public void computeStats(Analyzer analyzer) {
         super.computeStats(analyzer);
         if (!analyzer.safeIsEnableJoinReorderBasedCost()) {
@@ -262,41 +319,6 @@ public abstract class SetOperationNode extends PlanNode {
         Preconditions.checkState(conjuncts.isEmpty());
         computeTupleStatAndMemLayout(analyzer);
         computeStats(analyzer);
-        // except Node must not reorder the child
-        if (!(this instanceof ExceptNode)) {
-            computePassthrough(analyzer);
-        }
-        // drop resultExprs/constExprs that aren't getting materialized (= where the
-        // corresponding output slot isn't being materialized)
-        materializedResultExprLists_.clear();
-        Preconditions.checkState(resultExprLists_.size() == children.size());
-        List<SlotDescriptor> slots = analyzer.getDescTbl().getTupleDesc(tupleId_).getSlots();
-        for (int i = 0; i < resultExprLists_.size(); ++i) {
-            List<Expr> exprList = resultExprLists_.get(i);
-            List<Expr> newExprList = Lists.newArrayList();
-            Preconditions.checkState(exprList.size() == slots.size());
-            for (int j = 0; j < exprList.size(); ++j) {
-                if (slots.get(j).isMaterialized()) {
-                    newExprList.add(exprList.get(j));
-                }
-            }
-            materializedResultExprLists_.add(
-                    Expr.substituteList(newExprList, getChild(i).getOutputSmap(), analyzer, true));
-        }
-        Preconditions.checkState(
-                materializedResultExprLists_.size() == getChildren().size());
-
-        materializedConstExprLists_.clear();
-        for (List<Expr> exprList : constExprLists_) {
-            Preconditions.checkState(exprList.size() == slots.size());
-            List<Expr> newExprList = Lists.newArrayList();
-            for (int i = 0; i < exprList.size(); ++i) {
-                if (slots.get(i).isMaterialized()) {
-                    newExprList.add(exprList.get(i));
-                }
-            }
-            materializedConstExprLists_.add(newExprList);
-        }
     }
 
     protected void toThrift(TPlanNode msg, TPlanNodeType nodeType) {
diff --git a/fe/fe-core/src/test/java/org/apache/doris/planner/PlannerTest.java b/fe/fe-core/src/test/java/org/apache/doris/planner/PlannerTest.java
index b145917..34904a5 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/planner/PlannerTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/planner/PlannerTest.java
@@ -251,6 +251,28 @@ public class PlannerTest {
                 .getPlanRoot().getChild(0) instanceof AggregationNode);
         Assert.assertTrue(fragments10.get(0).getPlanRoot()
                 .getFragment().getPlanRoot().getChild(1) instanceof UnionNode);
+
+        String sql11 = "SELECT a.x FROM\n" +
+                "(SELECT '01' x) a \n" +
+                "INNER JOIN\n" +
+                "(SELECT '01' x UNION all SELECT '02') b";
+        StmtExecutor stmtExecutor11 = new StmtExecutor(ctx, sql11);
+        stmtExecutor11.execute();
+        Planner planner11 = stmtExecutor11.planner();
+        SetOperationNode setNode11 = (SetOperationNode)(planner11.getFragments().get(1).getPlanRoot());
+        Assert.assertEquals(2, setNode11.getMaterializedConstExprLists_().size());
+
+        String sql12 = "SELECT a.x \n" +
+                "FROM (SELECT '01' x) a \n" +
+                "INNER JOIN \n" +
+                "(SELECT k1 from db1.tbl1 \n" +
+                "UNION all \n" +
+                "SELECT k1 from db1.tbl1) b;";
+        StmtExecutor stmtExecutor12 = new StmtExecutor(ctx, sql12);
+        stmtExecutor12.execute();
+        Planner planner12 = stmtExecutor12.planner();
+        SetOperationNode setNode12 = (SetOperationNode)(planner12.getFragments().get(1).getPlanRoot());
+        Assert.assertEquals(2, setNode12.getMaterializedResultExprLists_().size());
     }
 
     @Test

---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org
For additional commands, e-mail: commits-help@doris.apache.org