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

morrysnow pushed a commit to branch branch-2.1
in repository https://gitbox.apache.org/repos/asf/doris.git


The following commit(s) were added to refs/heads/branch-2.1 by this push:
     new 10f3e88f7a2 [fix](nereids) fix distribution expr list (#39435)
10f3e88f7a2 is described below

commit 10f3e88f7a2495eda9608ba43420cbeeae3c93cb
Author: xzj7019 <131111794+xzj7...@users.noreply.github.com>
AuthorDate: Thu Aug 22 15:19:51 2024 +0800

    [fix](nereids) fix distribution expr list (#39435)
    
    pick from #39148
---
 .../org/apache/doris/nereids/NereidsPlanner.java   |   3 +-
 .../properties/ChildOutputPropertyDeriver.java     | 104 +++++++----
 .../properties/ChildOutputPropertyDeriverTest.java |  36 ++--
 .../data/nereids_p0/hint/test_distribute.out       | 193 ++++++++++-----------
 4 files changed, 188 insertions(+), 148 deletions(-)

diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/NereidsPlanner.java 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/NereidsPlanner.java
index 74a54036f9c..33c1554b1c8 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/NereidsPlanner.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/NereidsPlanner.java
@@ -434,8 +434,7 @@ public class NereidsPlanner extends Planner {
             // add groupExpression to plan so that we could print group id in 
plan.treeString()
             plan = plan.withGroupExpression(Optional.of(groupExpression));
             PhysicalPlan physicalPlan = ((PhysicalPlan) 
plan).withPhysicalPropertiesAndStats(
-                    groupExpression.getOutputProperties(physicalProperties),
-                    groupExpression.getOwnerGroup().getStatistics());
+                    physicalProperties, 
groupExpression.getOwnerGroup().getStatistics());
             return physicalPlan;
         } catch (Exception e) {
             if (e instanceof AnalysisException && 
e.getMessage().contains("Failed to choose best plan")) {
diff --git 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriver.java
 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriver.java
index bb23f46cdc4..270712496e9 100644
--- 
a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriver.java
+++ 
b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriver.java
@@ -256,8 +256,25 @@ public class ChildOutputPropertyDeriver extends 
PlanVisitor<PhysicalProperties,
 
         // broadcast
         if (rightOutputProperty.getDistributionSpec() instanceof 
DistributionSpecReplicated) {
-            DistributionSpec parentDistributionSpec = 
leftOutputProperty.getDistributionSpec();
-            return new PhysicalProperties(parentDistributionSpec);
+            DistributionSpec leftDistributionSpec = 
leftOutputProperty.getDistributionSpec();
+            // if left side is hash distribute and the key can satisfy the 
join keys, then mock
+            // a right side hash spec with the corresponding join keys, to 
filling the returning spec
+            // with refined EquivalenceExprIds.
+            if (leftDistributionSpec instanceof DistributionSpecHash
+                    && !(hashJoin.isMarkJoin() && 
hashJoin.getHashJoinConjuncts().isEmpty())
+                    && !hashJoin.getHashConjunctsExprIds().first.isEmpty()
+                    && !hashJoin.getHashConjunctsExprIds().second.isEmpty()
+                    && hashJoin.getHashConjunctsExprIds().first.size()
+                        == hashJoin.getHashConjunctsExprIds().second.size()
+                    && leftDistributionSpec.satisfy(
+                            new 
DistributionSpecHash(hashJoin.getHashConjunctsExprIds().first, 
ShuffleType.REQUIRE))) {
+                DistributionSpecHash mockedRightHashSpec = 
mockAnotherSideSpecFromConjuncts(
+                        hashJoin, (DistributionSpecHash) leftDistributionSpec);
+                return computeShuffleJoinOutputProperties(hashJoin,
+                        (DistributionSpecHash) leftDistributionSpec, 
mockedRightHashSpec);
+            } else {
+                return new PhysicalProperties(leftDistributionSpec);
+            }
         }
 
         // shuffle
@@ -265,33 +282,7 @@ public class ChildOutputPropertyDeriver extends 
PlanVisitor<PhysicalProperties,
                 && rightOutputProperty.getDistributionSpec() instanceof 
DistributionSpecHash) {
             DistributionSpecHash leftHashSpec = (DistributionSpecHash) 
leftOutputProperty.getDistributionSpec();
             DistributionSpecHash rightHashSpec = (DistributionSpecHash) 
rightOutputProperty.getDistributionSpec();
-
-            switch (hashJoin.getJoinType()) {
-                case INNER_JOIN:
-                case CROSS_JOIN:
-                    return new PhysicalProperties(DistributionSpecHash.merge(
-                            leftHashSpec, rightHashSpec, 
leftHashSpec.getShuffleType()));
-                case LEFT_SEMI_JOIN:
-                case LEFT_ANTI_JOIN:
-                case NULL_AWARE_LEFT_ANTI_JOIN:
-                case LEFT_OUTER_JOIN:
-                    return new PhysicalProperties(leftHashSpec);
-                case RIGHT_SEMI_JOIN:
-                case RIGHT_ANTI_JOIN:
-                case RIGHT_OUTER_JOIN:
-                    if (JoinUtils.couldColocateJoin(leftHashSpec, 
rightHashSpec, hashJoin.getHashJoinConjuncts())) {
-                        return new PhysicalProperties(rightHashSpec);
-                    } else {
-                        // retain left shuffle type, since coordinator use 
left most node to schedule fragment
-                        // forbid colocate join, since right table already 
shuffle
-                        return new 
PhysicalProperties(rightHashSpec.withShuffleTypeAndForbidColocateJoin(
-                                leftHashSpec.getShuffleType()));
-                    }
-                case FULL_OUTER_JOIN:
-                    return PhysicalProperties.createAnyFromHash(leftHashSpec);
-                default:
-                    throw new AnalysisException("unknown join type " + 
hashJoin.getJoinType());
-            }
+            return computeShuffleJoinOutputProperties(hashJoin, leftHashSpec, 
rightHashSpec);
         }
 
         throw new RuntimeException("Could not derive hash join's output 
properties. join: " + hashJoin);
@@ -465,6 +456,61 @@ public class ChildOutputPropertyDeriver extends 
PlanVisitor<PhysicalProperties,
         return childrenOutputProperties.get(0);
     }
 
+    private PhysicalProperties computeShuffleJoinOutputProperties(
+            PhysicalHashJoin<? extends Plan, ? extends Plan> hashJoin,
+            DistributionSpecHash leftHashSpec, DistributionSpecHash 
rightHashSpec) {
+        switch (hashJoin.getJoinType()) {
+            case INNER_JOIN:
+            case CROSS_JOIN:
+                return new PhysicalProperties(DistributionSpecHash.merge(
+                        leftHashSpec, rightHashSpec, 
leftHashSpec.getShuffleType()));
+            case LEFT_SEMI_JOIN:
+            case LEFT_ANTI_JOIN:
+            case NULL_AWARE_LEFT_ANTI_JOIN:
+            case LEFT_OUTER_JOIN:
+                return new PhysicalProperties(leftHashSpec);
+            case RIGHT_SEMI_JOIN:
+            case RIGHT_ANTI_JOIN:
+            case RIGHT_OUTER_JOIN:
+                if (JoinUtils.couldColocateJoin(leftHashSpec, rightHashSpec, 
hashJoin.getHashJoinConjuncts())) {
+                    return new PhysicalProperties(rightHashSpec);
+                } else {
+                    // retain left shuffle type, since coordinator use left 
most node to schedule fragment
+                    // forbid colocate join, since right table already shuffle
+                    return new 
PhysicalProperties(rightHashSpec.withShuffleTypeAndForbidColocateJoin(
+                            leftHashSpec.getShuffleType()));
+                }
+            case FULL_OUTER_JOIN:
+                return PhysicalProperties.createAnyFromHash(leftHashSpec);
+            default:
+                throw new AnalysisException("unknown join type " + 
hashJoin.getJoinType());
+        }
+    }
+
+    private DistributionSpecHash mockAnotherSideSpecFromConjuncts(
+            PhysicalHashJoin<? extends Plan, ? extends Plan> hashJoin, 
DistributionSpecHash oneSideSpec) {
+        List<ExprId> leftExprIds = hashJoin.getHashConjunctsExprIds().first;
+        List<ExprId> rightExprIds = hashJoin.getHashConjunctsExprIds().second;
+        Preconditions.checkState(!leftExprIds.isEmpty() && 
!rightExprIds.isEmpty()
+                && leftExprIds.size() == rightExprIds.size(), "invalid hash 
join conjuncts");
+        List<ExprId> anotherSideOrderedExprIds = Lists.newArrayList();
+        for (ExprId exprId : oneSideSpec.getOrderedShuffledColumns()) {
+            int index = leftExprIds.indexOf(exprId);
+            if (index == -1) {
+                Set<ExprId> equivalentExprIds = 
oneSideSpec.getEquivalenceExprIdsOf(exprId);
+                for (ExprId id : equivalentExprIds) {
+                    index = leftExprIds.indexOf(id);
+                    if (index >= 0) {
+                        break;
+                    }
+                }
+                Preconditions.checkState(index >= 0, "can't find exprId in 
equivalence set");
+            }
+            anotherSideOrderedExprIds.add(rightExprIds.get(index));
+        }
+        return new DistributionSpecHash(anotherSideOrderedExprIds, 
oneSideSpec.getShuffleType());
+    }
+
     private boolean isSameHashValue(DataType originType, DataType castType) {
         if (originType.isStringLikeType() && (castType.isVarcharType() || 
castType.isStringType())
                 && (castType.width() >= originType.width() || castType.width() 
< 0)) {
diff --git 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriverTest.java
 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriverTest.java
index e6a7e601c24..33ea59342d3 100644
--- 
a/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriverTest.java
+++ 
b/fe/fe-core/src/test/java/org/apache/doris/nereids/properties/ChildOutputPropertyDeriverTest.java
@@ -19,14 +19,17 @@ package org.apache.doris.nereids.properties;
 
 import org.apache.doris.catalog.ColocateTableIndex;
 import org.apache.doris.catalog.Env;
+import org.apache.doris.common.IdGenerator;
 import org.apache.doris.common.Pair;
 import org.apache.doris.nereids.hint.DistributeHint;
 import org.apache.doris.nereids.memo.Group;
 import org.apache.doris.nereids.memo.GroupExpression;
+import org.apache.doris.nereids.memo.GroupId;
 import org.apache.doris.nereids.properties.DistributionSpecHash.ShuffleType;
 import org.apache.doris.nereids.trees.expressions.AssertNumRowsElement;
 import org.apache.doris.nereids.trees.expressions.EqualTo;
 import org.apache.doris.nereids.trees.expressions.ExprId;
+import org.apache.doris.nereids.trees.expressions.Slot;
 import org.apache.doris.nereids.trees.expressions.SlotReference;
 import org.apache.doris.nereids.trees.expressions.functions.agg.AggregateParam;
 import org.apache.doris.nereids.trees.plans.AggMode;
@@ -63,6 +66,7 @@ import org.junit.jupiter.api.Assertions;
 import org.junit.jupiter.api.BeforeEach;
 import org.junit.jupiter.api.Test;
 
+import java.util.ArrayList;
 import java.util.Collections;
 import java.util.List;
 import java.util.Map;
@@ -529,21 +533,25 @@ class ChildOutputPropertyDeriverTest {
     }
 
     @Test
-    void testBroadcastJoin() {
-        new MockUp<JoinUtils>() {
-            @Mock
-            Pair<List<ExprId>, List<ExprId>> getOnClauseUsedSlots(
-                    AbstractPhysicalJoin<? extends Plan, ? extends Plan> join) 
{
-                return Pair.of(Lists.newArrayList(new ExprId(0)), 
Lists.newArrayList(new ExprId(2)));
-            }
-        };
-
+    void testBroadcastJoin(@Injectable LogicalProperties p1, @Injectable 
GroupPlan p2) {
+        SlotReference leftSlot = new SlotReference(new ExprId(0), "left", 
IntegerType.INSTANCE, false, Collections.emptyList());
+        SlotReference rightSlot = new SlotReference(new ExprId(2), "right", 
IntegerType.INSTANCE, false, Collections.emptyList());
+        List<Slot> leftOutput = new ArrayList<>();
+        List<Slot> rightOutput = new ArrayList<>();
+        leftOutput.add(leftSlot);
+        rightOutput.add(rightSlot);
+        LogicalProperties leftProperties = new LogicalProperties(() -> 
leftOutput, () -> FunctionalDependencies.EMPTY_FUNC_DEPS);
+        LogicalProperties rightProperties = new LogicalProperties(() -> 
rightOutput, () -> FunctionalDependencies.EMPTY_FUNC_DEPS);
+
+        IdGenerator<GroupId> idGenerator = GroupId.createGenerator();
+        GroupPlan leftGroupPlan = new GroupPlan(new 
Group(idGenerator.getNextId(), leftProperties));
+        GroupPlan rightGroupPlan = new GroupPlan(new 
Group(idGenerator.getNextId(), rightProperties));
         PhysicalHashJoin<GroupPlan, GroupPlan> join = new 
PhysicalHashJoin<>(JoinType.INNER_JOIN,
                 Lists.newArrayList(new EqualTo(
-                        new SlotReference(new ExprId(0), "left", 
IntegerType.INSTANCE, false, Collections.emptyList()),
-                        new SlotReference(new ExprId(2), "right", 
IntegerType.INSTANCE, false,
-                                Collections.emptyList()))),
-                ExpressionUtils.EMPTY_CONDITION, new 
DistributeHint(DistributeType.NONE), Optional.empty(), logicalProperties, 
groupPlan, groupPlan);
+                        leftSlot, rightSlot
+                        )),
+                ExpressionUtils.EMPTY_CONDITION, new 
DistributeHint(DistributeType.NONE),
+                Optional.empty(), logicalProperties, leftGroupPlan, 
rightGroupPlan);
         GroupExpression groupExpression = new GroupExpression(join);
         new Group(null, groupExpression, null);
 
@@ -572,7 +580,7 @@ class ChildOutputPropertyDeriverTest {
         DistributionSpecHash actual = (DistributionSpecHash) 
result.getDistributionSpec();
         Assertions.assertEquals(ShuffleType.NATURAL, actual.getShuffleType());
         // check merged
-        Assertions.assertEquals(2, actual.getExprIdToEquivalenceSet().size());
+        Assertions.assertEquals(3, actual.getExprIdToEquivalenceSet().size());
     }
 
     @Test
diff --git a/regression-test/data/nereids_p0/hint/test_distribute.out 
b/regression-test/data/nereids_p0/hint/test_distribute.out
index 443a1f77efa..5379183bebe 100644
--- a/regression-test/data/nereids_p0/hint/test_distribute.out
+++ b/regression-test/data/nereids_p0/hint/test_distribute.out
@@ -143,7 +143,7 @@ PhysicalResultSink
 ----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t2]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t3]
 
@@ -159,14 +159,13 @@ PhysicalResultSink
 ------hashAgg[LOCAL]
 --------PhysicalProject
 ----------hashJoin[INNER_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-------------PhysicalDistribute[DistributionSpecHash]
---------------PhysicalProject
-----------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+------------PhysicalProject
+--------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+----------------PhysicalProject
+------------------PhysicalOlapScan[t1]
+----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
---------------------PhysicalOlapScan[t1]
-------------------PhysicalDistribute[DistributionSpecReplicated]
---------------------PhysicalProject
-----------------------PhysicalOlapScan[t2]
+--------------------PhysicalOlapScan[t2]
 ------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t3]
@@ -289,7 +288,7 @@ PhysicalResultSink
 --------------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------------PhysicalProject
 ------------------------PhysicalOlapScan[t2]
-----------------PhysicalDistribute[DistributionSpecReplicated]
+----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecReplicated]
@@ -320,7 +319,7 @@ PhysicalResultSink
 ----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t4]
 
@@ -366,14 +365,13 @@ PhysicalResultSink
 ----------hashJoin[INNER_JOIN] hashCondition=((t3.c3 = t4.c4)) 
otherCondition=()
 ------------PhysicalProject
 --------------hashJoin[INNER_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-----------------PhysicalDistribute[DistributionSpecHash]
-------------------PhysicalProject
---------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+----------------PhysicalProject
+------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+--------------------PhysicalProject
+----------------------PhysicalOlapScan[t1]
+--------------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------------PhysicalProject
-------------------------PhysicalOlapScan[t1]
-----------------------PhysicalDistribute[DistributionSpecReplicated]
-------------------------PhysicalProject
---------------------------PhysicalOlapScan[t2]
+------------------------PhysicalOlapScan[t2]
 ----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
@@ -845,7 +843,7 @@ PhysicalResultSink
 ------------------PhysicalProject
 --------------------filter((t2.c2 <= 300) and (t2.c2 >= 100))
 ----------------------PhysicalOlapScan[t2]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------filter((t3.c3 <= 300) and (t3.c3 >= 100))
 ------------------PhysicalOlapScan[t3]
@@ -871,7 +869,7 @@ PhysicalResultSink
 ------------------PhysicalProject
 --------------------filter((t2.c2 <= 300) and (t2.c2 >= 100))
 ----------------------PhysicalOlapScan[t2]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------filter((t3.c3 <= 300) and (t3.c3 >= 100))
 ------------------PhysicalOlapScan[t3]
@@ -921,7 +919,7 @@ PhysicalResultSink
 ----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t2]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t3]
 
@@ -1196,7 +1194,7 @@ PhysicalResultSink
 ----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t2]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t3]
 
@@ -1350,14 +1348,13 @@ PhysicalResultSink
 ------hashAgg[LOCAL]
 --------PhysicalProject
 ----------hashJoin[LEFT_OUTER_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-------------PhysicalDistribute[DistributionSpecHash]
---------------PhysicalProject
-----------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+------------PhysicalProject
+--------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+----------------PhysicalProject
+------------------PhysicalOlapScan[t1]
+----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
---------------------PhysicalOlapScan[t1]
-------------------PhysicalDistribute[DistributionSpecReplicated]
---------------------PhysicalProject
-----------------------PhysicalOlapScan[t2]
+--------------------PhysicalOlapScan[t2]
 ------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t3]
@@ -1605,7 +1602,7 @@ PhysicalResultSink
 --------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------PhysicalProject
 ------------------PhysicalOlapScan[t2]
-----------PhysicalDistribute[DistributionSpecReplicated]
+----------PhysicalDistribute[DistributionSpecHash]
 ------------PhysicalProject
 --------------PhysicalOlapScan[t3]
 
@@ -1752,14 +1749,13 @@ PhysicalResultSink
 ----PhysicalDistribute[DistributionSpecGather]
 ------hashAgg[LOCAL]
 --------hashJoin[LEFT_SEMI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-----------PhysicalDistribute[DistributionSpecHash]
-------------PhysicalProject
---------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+----------PhysicalProject
+------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+--------------PhysicalProject
+----------------PhysicalOlapScan[t1]
+--------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------PhysicalProject
-------------------PhysicalOlapScan[t1]
-----------------PhysicalDistribute[DistributionSpecReplicated]
-------------------PhysicalProject
---------------------PhysicalOlapScan[t2]
+------------------PhysicalOlapScan[t2]
 ----------PhysicalDistribute[DistributionSpecHash]
 ------------PhysicalProject
 --------------PhysicalOlapScan[t3]
@@ -1799,7 +1795,7 @@ PhysicalResultSink
 --------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------PhysicalProject
 ------------------PhysicalOlapScan[t2]
-----------PhysicalDistribute[DistributionSpecReplicated]
+----------PhysicalDistribute[DistributionSpecHash]
 ------------PhysicalProject
 --------------PhysicalOlapScan[t3]
 
@@ -1946,14 +1942,13 @@ PhysicalResultSink
 ----PhysicalDistribute[DistributionSpecGather]
 ------hashAgg[LOCAL]
 --------hashJoin[LEFT_ANTI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-----------PhysicalDistribute[DistributionSpecHash]
-------------PhysicalProject
---------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+----------PhysicalProject
+------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+--------------PhysicalProject
+----------------PhysicalOlapScan[t1]
+--------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------PhysicalProject
-------------------PhysicalOlapScan[t1]
-----------------PhysicalDistribute[DistributionSpecReplicated]
-------------------PhysicalProject
---------------------PhysicalOlapScan[t2]
+------------------PhysicalOlapScan[t2]
 ----------PhysicalDistribute[DistributionSpecHash]
 ------------PhysicalProject
 --------------PhysicalOlapScan[t3]
@@ -2406,7 +2401,7 @@ PhysicalResultSink
 --------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------PhysicalProject
 ------------------PhysicalOlapScan[t2]
-----------PhysicalDistribute[DistributionSpecReplicated]
+----------PhysicalDistribute[DistributionSpecHash]
 ------------PhysicalProject
 --------------PhysicalOlapScan[t3]
 
@@ -2553,14 +2548,13 @@ PhysicalResultSink
 ----PhysicalDistribute[DistributionSpecGather]
 ------hashAgg[LOCAL]
 --------hashJoin[LEFT_SEMI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-----------PhysicalDistribute[DistributionSpecHash]
-------------PhysicalProject
---------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+----------PhysicalProject
+------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+--------------PhysicalProject
+----------------PhysicalOlapScan[t1]
+--------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------PhysicalProject
-------------------PhysicalOlapScan[t1]
-----------------PhysicalDistribute[DistributionSpecReplicated]
-------------------PhysicalProject
---------------------PhysicalOlapScan[t2]
+------------------PhysicalOlapScan[t2]
 ----------PhysicalDistribute[DistributionSpecHash]
 ------------PhysicalProject
 --------------PhysicalOlapScan[t3]
@@ -3497,14 +3491,13 @@ PhysicalResultSink
 ----------hashJoin[INNER_JOIN] hashCondition=((t3.c3 = t4.c4)) 
otherCondition=()
 ------------PhysicalProject
 --------------hashJoin[INNER_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-----------------PhysicalDistribute[DistributionSpecHash]
-------------------PhysicalProject
---------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+----------------PhysicalProject
+------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+--------------------PhysicalProject
+----------------------PhysicalOlapScan[t1]
+--------------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------------PhysicalProject
-------------------------PhysicalOlapScan[t1]
-----------------------PhysicalDistribute[DistributionSpecReplicated]
-------------------------PhysicalProject
---------------------------PhysicalOlapScan[t2]
+------------------------PhysicalOlapScan[t2]
 ----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
@@ -3536,7 +3529,7 @@ PhysicalResultSink
 ----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t4]
 
@@ -3589,7 +3582,7 @@ PhysicalResultSink
 --------------------PhysicalDistribute[DistributionSpecReplicated]
 ----------------------PhysicalProject
 ------------------------PhysicalOlapScan[t2]
-----------------PhysicalDistribute[DistributionSpecReplicated]
+----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecReplicated]
@@ -3620,7 +3613,7 @@ PhysicalResultSink
 ----------------PhysicalDistribute[DistributionSpecReplicated]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
-------------PhysicalDistribute[DistributionSpecReplicated]
+------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t4]
 
@@ -4226,19 +4219,17 @@ PhysicalResultSink
 ------hashAgg[LOCAL]
 --------PhysicalProject
 ----------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t4.c4)) 
otherCondition=()
-------------PhysicalDistribute[DistributionSpecHash]
---------------PhysicalProject
-----------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-------------------PhysicalDistribute[DistributionSpecHash]
---------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
-----------------------PhysicalProject
-------------------------PhysicalOlapScan[t1]
-----------------------PhysicalDistribute[DistributionSpecReplicated]
-------------------------PhysicalProject
---------------------------PhysicalOlapScan[t2]
-------------------PhysicalDistribute[DistributionSpecHash]
+------------PhysicalProject
+--------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
+----------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+------------------PhysicalProject
+--------------------PhysicalOlapScan[t1]
+------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
-----------------------PhysicalOlapScan[t3]
+----------------------PhysicalOlapScan[t2]
+----------------PhysicalDistribute[DistributionSpecHash]
+------------------PhysicalProject
+--------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t4]
@@ -4317,7 +4308,7 @@ PhysicalResultSink
 ------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
 ----------------------PhysicalOlapScan[t2]
-----------------PhysicalDistribute[DistributionSpecReplicated]
+----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecReplicated]
@@ -4934,19 +4925,17 @@ PhysicalResultSink
 ------hashAgg[LOCAL]
 --------PhysicalProject
 ----------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t4.c4)) 
otherCondition=()
-------------PhysicalDistribute[DistributionSpecHash]
---------------PhysicalProject
-----------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-------------------PhysicalDistribute[DistributionSpecHash]
---------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
-----------------------PhysicalProject
-------------------------PhysicalOlapScan[t1]
-----------------------PhysicalDistribute[DistributionSpecReplicated]
-------------------------PhysicalProject
---------------------------PhysicalOlapScan[t2]
-------------------PhysicalDistribute[DistributionSpecHash]
+------------PhysicalProject
+--------------hashJoin[LEFT_SEMI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
+----------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+------------------PhysicalProject
+--------------------PhysicalOlapScan[t1]
+------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
-----------------------PhysicalOlapScan[t3]
+----------------------PhysicalOlapScan[t2]
+----------------PhysicalDistribute[DistributionSpecHash]
+------------------PhysicalProject
+--------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t4]
@@ -5025,7 +5014,7 @@ PhysicalResultSink
 ------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
 ----------------------PhysicalOlapScan[t2]
-----------------PhysicalDistribute[DistributionSpecReplicated]
+----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecReplicated]
@@ -5287,19 +5276,17 @@ PhysicalResultSink
 ------hashAgg[LOCAL]
 --------PhysicalProject
 ----------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t4.c4)) 
otherCondition=()
-------------PhysicalDistribute[DistributionSpecHash]
---------------PhysicalProject
-----------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
-------------------PhysicalDistribute[DistributionSpecHash]
---------------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
-----------------------PhysicalProject
-------------------------PhysicalOlapScan[t1]
-----------------------PhysicalDistribute[DistributionSpecReplicated]
-------------------------PhysicalProject
---------------------------PhysicalOlapScan[t2]
-------------------PhysicalDistribute[DistributionSpecHash]
+------------PhysicalProject
+--------------hashJoin[LEFT_ANTI_JOIN] hashCondition=((t2.c2 = t3.c3)) 
otherCondition=()
+----------------hashJoin[INNER_JOIN] hashCondition=((t1.c1 = t2.c2)) 
otherCondition=()
+------------------PhysicalProject
+--------------------PhysicalOlapScan[t1]
+------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
-----------------------PhysicalOlapScan[t3]
+----------------------PhysicalOlapScan[t2]
+----------------PhysicalDistribute[DistributionSpecHash]
+------------------PhysicalProject
+--------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecHash]
 --------------PhysicalProject
 ----------------PhysicalOlapScan[t4]
@@ -5378,7 +5365,7 @@ PhysicalResultSink
 ------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
 ----------------------PhysicalOlapScan[t2]
-----------------PhysicalDistribute[DistributionSpecReplicated]
+----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecReplicated]
@@ -5881,7 +5868,7 @@ PhysicalResultSink
 ------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
 ----------------------PhysicalOlapScan[t2]
-----------------PhysicalDistribute[DistributionSpecReplicated]
+----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecReplicated]
@@ -7247,7 +7234,7 @@ PhysicalResultSink
 ------------------PhysicalDistribute[DistributionSpecReplicated]
 --------------------PhysicalProject
 ----------------------PhysicalOlapScan[t2]
-----------------PhysicalDistribute[DistributionSpecReplicated]
+----------------PhysicalDistribute[DistributionSpecHash]
 ------------------PhysicalProject
 --------------------PhysicalOlapScan[t3]
 ------------PhysicalDistribute[DistributionSpecReplicated]


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

Reply via email to