This is an automated email from the ASF dual-hosted git repository. jakevin pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/doris.git
The following commit(s) were added to refs/heads/master by this push: new bd46d721e9 [feature](Nereids): pull up SEMI JOIN from INNER JOIN (#17765) bd46d721e9 is described below commit bd46d721e9ca353841f9b1ec0dd897d4c337720d Author: jakevin <jakevin...@gmail.com> AuthorDate: Wed Mar 22 12:48:04 2023 +0800 [feature](Nereids): pull up SEMI JOIN from INNER JOIN (#17765) --- .../org/apache/doris/nereids/rules/RuleSet.java | 4 ++ .../join/LogicalJoinSemiJoinTranspose.java | 52 ++++++++++++++ .../join/LogicalJoinSemiJoinTransposeProject.java | 57 +++++++++++++++ .../join/SemiJoinLogicalJoinTranspose.java | 48 ++++--------- .../join/SemiJoinLogicalJoinTransposeProject.java | 11 ++- .../nereids/trees/plans/logical/LogicalJoin.java | 10 ++- .../LogicalJoinSemiJoinTransposeProjectTest.java | 83 ++++++++++++++++++++++ .../join/LogicalJoinSemiJoinTransposeTest.java | 56 +++++++++++++++ .../LogicalWindowToPhysicalWindowTest.java | 3 +- 9 files changed, 278 insertions(+), 46 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java index 11b98fe257..ada9c98ad3 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java @@ -20,6 +20,8 @@ package org.apache.doris.nereids.rules; import org.apache.doris.nereids.rules.exploration.join.InnerJoinLAsscom; import org.apache.doris.nereids.rules.exploration.join.InnerJoinLAsscomProject; import org.apache.doris.nereids.rules.exploration.join.JoinCommute; +import org.apache.doris.nereids.rules.exploration.join.LogicalJoinSemiJoinTranspose; +import org.apache.doris.nereids.rules.exploration.join.LogicalJoinSemiJoinTransposeProject; import org.apache.doris.nereids.rules.exploration.join.OuterJoinLAsscom; import org.apache.doris.nereids.rules.exploration.join.OuterJoinLAsscomProject; import org.apache.doris.nereids.rules.exploration.join.PushdownProjectThroughInnerJoin; @@ -88,6 +90,8 @@ public class RuleSet { .add(SemiJoinLogicalJoinTransposeProject.LEFT_DEEP) .add(SemiJoinSemiJoinTranspose.INSTANCE) .add(SemiJoinSemiJoinTransposeProject.INSTANCE) + .add(LogicalJoinSemiJoinTranspose.INSTANCE) + .add(LogicalJoinSemiJoinTransposeProject.INSTANCE) .add(PushdownProjectThroughInnerJoin.INSTANCE) .add(PushdownProjectThroughSemiJoin.INSTANCE) .build(); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTranspose.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTranspose.java new file mode 100644 index 0000000000..e9c9937f0f --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTranspose.java @@ -0,0 +1,52 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you under the Apache License, Version 2.0 (the +// "License"); you may not use this file except in compliance +// with the License. You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, +// software distributed under the License is distributed on an +// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY +// KIND, either express or implied. See the License for the +// specific language governing permissions and limitations +// under the License. + +package org.apache.doris.nereids.rules.exploration.join; + +import org.apache.doris.nereids.rules.Rule; +import org.apache.doris.nereids.rules.RuleType; +import org.apache.doris.nereids.rules.exploration.OneExplorationRuleFactory; +import org.apache.doris.nereids.trees.plans.GroupPlan; +import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.logical.LogicalJoin; + +/** + * LogicalJoin(SemiJoin(A, B), C) -> SemiJoin(LogicalJoin(A, C), B) + */ +public class LogicalJoinSemiJoinTranspose extends OneExplorationRuleFactory { + + public static final LogicalJoinSemiJoinTranspose INSTANCE = new LogicalJoinSemiJoinTranspose(); + + @Override + public Rule build() { + return logicalJoin(logicalJoin(), group()) + .when(topJoin -> (topJoin.left().getJoinType().isLeftSemiOrAntiJoin() + && (topJoin.getJoinType().isInnerJoin() + || topJoin.getJoinType().isLeftOuterJoin()))) + .whenNot(topJoin -> topJoin.hasJoinHint() || topJoin.left().hasJoinHint()) + .whenNot(LogicalJoin::isMarkJoin) + .then(topJoin -> { + LogicalJoin<GroupPlan, GroupPlan> bottomJoin = topJoin.left(); + GroupPlan a = bottomJoin.left(); + GroupPlan b = bottomJoin.right(); + GroupPlan c = topJoin.right(); + + Plan newBottomJoin = topJoin.withChildren(a, c); + return bottomJoin.withChildren(newBottomJoin, b); + }).toRule(RuleType.LOGICAL_JOIN_LOGICAL_SEMI_JOIN_TRANSPOSE); + } +} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeProject.java new file mode 100644 index 0000000000..25ccafc40f --- /dev/null +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeProject.java @@ -0,0 +1,57 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you under the Apache License, Version 2.0 (the +// "License"); you may not use this file except in compliance +// with the License. You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, +// software distributed under the License is distributed on an +// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY +// KIND, either express or implied. See the License for the +// specific language governing permissions and limitations +// under the License. + +package org.apache.doris.nereids.rules.exploration.join; + +import org.apache.doris.nereids.rules.Rule; +import org.apache.doris.nereids.rules.RuleType; +import org.apache.doris.nereids.rules.exploration.OneExplorationRuleFactory; +import org.apache.doris.nereids.trees.plans.GroupPlan; +import org.apache.doris.nereids.trees.plans.Plan; +import org.apache.doris.nereids.trees.plans.logical.LogicalJoin; + +import java.util.ArrayList; + +/** + * LogicalJoin(SemiJoin(A, B), C) -> SemiJoin(LogicalJoin(A, C), B) + */ +public class LogicalJoinSemiJoinTransposeProject extends OneExplorationRuleFactory { + + public static final LogicalJoinSemiJoinTransposeProject INSTANCE = new LogicalJoinSemiJoinTransposeProject(); + + @Override + public Rule build() { + return logicalJoin(logicalProject(logicalJoin()), group()) + .when(topJoin -> (topJoin.left().child().getJoinType().isLeftSemiOrAntiJoin() + && (topJoin.getJoinType().isInnerJoin() + || topJoin.getJoinType().isLeftOuterJoin()))) + .whenNot(topJoin -> topJoin.hasJoinHint() || topJoin.left().child().hasJoinHint()) + .whenNot(LogicalJoin::isMarkJoin) + .when(join -> JoinReorderUtils.isAllSlotProject(join.left())) + .then(topJoin -> { + LogicalJoin<GroupPlan, GroupPlan> bottomJoin = topJoin.left().child(); + GroupPlan a = bottomJoin.left(); + GroupPlan b = bottomJoin.right(); + GroupPlan c = topJoin.right(); + + // Discard this project, because it is useless. + Plan newBottomJoin = topJoin.withChildren(a, c); + Plan newTopJoin = bottomJoin.withChildren(newBottomJoin, b); + return JoinReorderUtils.projectOrSelfInOrder(new ArrayList<>(topJoin.getOutput()), newTopJoin); + }).toRule(RuleType.LOGICAL_JOIN_LOGICAL_SEMI_JOIN_TRANSPOSE_PROJECT); + } +} diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTranspose.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTranspose.java index 73f202cd26..158ff03abc 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTranspose.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTranspose.java @@ -20,17 +20,15 @@ package org.apache.doris.nereids.rules.exploration.join; import org.apache.doris.nereids.rules.Rule; import org.apache.doris.nereids.rules.RuleType; import org.apache.doris.nereids.rules.exploration.OneExplorationRuleFactory; +import org.apache.doris.nereids.rules.exploration.join.SemiJoinLogicalJoinTransposeProject.ContainsType; import org.apache.doris.nereids.trees.expressions.ExprId; -import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.plans.GroupPlan; import org.apache.doris.nereids.trees.plans.JoinType; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.logical.LogicalJoin; -import org.apache.doris.nereids.util.Utils; import com.google.common.base.Preconditions; -import java.util.List; import java.util.Set; /** @@ -56,9 +54,8 @@ public class SemiJoinLogicalJoinTranspose extends OneExplorationRuleFactory { return logicalJoin(logicalJoin(), group()) .when(topJoin -> (topJoin.getJoinType().isLeftSemiOrAntiJoin() && (topJoin.left().getJoinType().isInnerJoin() - || topJoin.left().getJoinType().isLeftOuterJoin() - || topJoin.left().getJoinType().isRightOuterJoin()))) - .when(this::conditionChecker) + || topJoin.left().getJoinType().isLeftOuterJoin() + || topJoin.left().getJoinType().isRightOuterJoin()))) .whenNot(topJoin -> topJoin.hasJoinHint() || topJoin.left().hasJoinHint()) .whenNot(LogicalJoin::isMarkJoin) .then(topSemiJoin -> { @@ -67,16 +64,13 @@ public class SemiJoinLogicalJoinTranspose extends OneExplorationRuleFactory { GroupPlan b = bottomJoin.right(); GroupPlan c = topSemiJoin.right(); - List<Expression> hashJoinConjuncts = topSemiJoin.getHashJoinConjuncts(); - Set<ExprId> aOutputExprIdSet = a.getOutputExprIdSet(); - - boolean lasscom = false; - for (Expression hashJoinConjunct : hashJoinConjuncts) { - Set<ExprId> usedSlotExprIds = hashJoinConjunct.getInputSlotExprIds(); - lasscom = Utils.isIntersecting(usedSlotExprIds, aOutputExprIdSet) || lasscom; + Set<ExprId> conjunctsIds = topSemiJoin.getConditionExprId(); + ContainsType containsType = SemiJoinLogicalJoinTransposeProject.containsChildren(conjunctsIds, + a.getOutputExprIdSet(), b.getOutputExprIdSet()); + if (containsType == ContainsType.ALL) { + return null; } - - if (lasscom) { + if (containsType == ContainsType.LEFT) { /* * topSemiJoin newTopJoin * / \ / \ @@ -90,6 +84,9 @@ public class SemiJoinLogicalJoinTranspose extends OneExplorationRuleFactory { Plan newBottomSemiJoin = topSemiJoin.withChildren(a, c); return bottomJoin.withChildren(newBottomSemiJoin, b); } else { + if (leftDeep) { + return null; + } /* * topSemiJoin newTopJoin * / \ / \ @@ -105,25 +102,4 @@ public class SemiJoinLogicalJoinTranspose extends OneExplorationRuleFactory { } }).toRule(RuleType.LOGICAL_SEMI_JOIN_LOGICAL_JOIN_TRANSPOSE); } - - // bottomJoin just return A OR B, else return false. - private boolean conditionChecker(LogicalJoin<LogicalJoin<GroupPlan, GroupPlan>, GroupPlan> topSemiJoin) { - List<Expression> hashJoinConjuncts = topSemiJoin.getHashJoinConjuncts(); - - Set<ExprId> aOutputExprIdSet = topSemiJoin.left().left().getOutputExprIdSet(); - Set<ExprId> bOutputExprIdSet = topSemiJoin.left().right().getOutputExprIdSet(); - - boolean hashContainsA = false; - boolean hashContainsB = false; - for (Expression hashJoinConjunct : hashJoinConjuncts) { - Set<ExprId> usedSlotExprIds = hashJoinConjunct.getInputSlotExprIds(); - hashContainsA = Utils.isIntersecting(usedSlotExprIds, aOutputExprIdSet) || hashContainsA; - hashContainsB = Utils.isIntersecting(usedSlotExprIds, bOutputExprIdSet) || hashContainsB; - } - if (leftDeep && hashContainsB) { - return false; - } - Preconditions.checkState(hashContainsA || hashContainsB, "join output must contain child"); - return !(hashContainsA && hashContainsB); - } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTransposeProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTransposeProject.java index a645ca3062..46aca17216 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTransposeProject.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/exploration/join/SemiJoinLogicalJoinTransposeProject.java @@ -31,8 +31,6 @@ import org.apache.doris.nereids.util.Utils; import com.google.common.base.Preconditions; import java.util.Set; -import java.util.stream.Collectors; -import java.util.stream.Stream; /** * <ul> @@ -68,9 +66,7 @@ public class SemiJoinLogicalJoinTransposeProject extends OneExplorationRuleFacto GroupPlan b = bottomJoin.right(); GroupPlan c = topSemiJoin.right(); - Set<ExprId> conjunctsIds = Stream.concat(topSemiJoin.getHashJoinConjuncts().stream(), - topSemiJoin.getOtherJoinConjuncts().stream()) - .flatMap(expr -> expr.getInputSlotExprIds().stream()).collect(Collectors.toSet()); + Set<ExprId> conjunctsIds = topSemiJoin.getConditionExprId(); ContainsType containsType = containsChildren(conjunctsIds, a.getOutputExprIdSet(), b.getOutputExprIdSet()); if (containsType == ContainsType.ALL) { @@ -120,7 +116,10 @@ public class SemiJoinLogicalJoinTransposeProject extends OneExplorationRuleFacto LEFT, RIGHT, ALL } - private ContainsType containsChildren(Set<ExprId> conjunctsExprIdSet, Set<ExprId> left, Set<ExprId> right) { + /** + * Check conjuncts contain children. + */ + public static ContainsType containsChildren(Set<ExprId> conjunctsExprIdSet, Set<ExprId> left, Set<ExprId> right) { boolean containsLeft = Utils.isIntersecting(conjunctsExprIdSet, left); boolean containsRight = Utils.isIntersecting(conjunctsExprIdSet, right); Preconditions.checkState(containsLeft || containsRight, "join output must contain child"); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalJoin.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalJoin.java index 7794e0bac0..1317e79a9d 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalJoin.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalJoin.java @@ -20,6 +20,7 @@ package org.apache.doris.nereids.trees.plans.logical; import org.apache.doris.nereids.memo.GroupExpression; import org.apache.doris.nereids.properties.LogicalProperties; import org.apache.doris.nereids.rules.exploration.join.JoinReorderContext; +import org.apache.doris.nereids.trees.expressions.ExprId; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.MarkJoinSlotReference; import org.apache.doris.nereids.trees.expressions.Slot; @@ -43,6 +44,7 @@ import java.util.List; import java.util.Objects; import java.util.Optional; import java.util.Set; +import java.util.stream.Collectors; import java.util.stream.Stream; /** @@ -135,8 +137,12 @@ public class LogicalJoin<LEFT_CHILD_TYPE extends Plan, RIGHT_CHILD_TYPE extends public Set<Slot> getConditionSlot() { return Stream.concat(hashJoinConjuncts.stream(), otherJoinConjuncts.stream()) - .flatMap(expr -> expr.getInputSlots().stream()) - .collect(ImmutableSet.toImmutableSet()); + .flatMap(expr -> expr.getInputSlots().stream()).collect(ImmutableSet.toImmutableSet()); + } + + public Set<ExprId> getConditionExprId() { + return Stream.concat(getHashJoinConjuncts().stream(), getOtherJoinConjuncts().stream()) + .flatMap(expr -> expr.getInputSlotExprIds().stream()).collect(Collectors.toSet()); } public Optional<Expression> getOnClauseCondition() { diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeProjectTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeProjectTest.java new file mode 100644 index 0000000000..a15b1845a8 --- /dev/null +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeProjectTest.java @@ -0,0 +1,83 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you under the Apache License, Version 2.0 (the +// "License"); you may not use this file except in compliance +// with the License. You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, +// software distributed under the License is distributed on an +// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY +// KIND, either express or implied. See the License for the +// specific language governing permissions and limitations +// under the License. + +package org.apache.doris.nereids.rules.exploration.join; + +import org.apache.doris.common.Pair; +import org.apache.doris.nereids.trees.plans.JoinType; +import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; +import org.apache.doris.nereids.trees.plans.logical.LogicalPlan; +import org.apache.doris.nereids.util.LogicalPlanBuilder; +import org.apache.doris.nereids.util.MemoPatternMatchSupported; +import org.apache.doris.nereids.util.MemoTestUtils; +import org.apache.doris.nereids.util.PlanChecker; +import org.apache.doris.nereids.util.PlanConstructor; + +import com.google.common.collect.ImmutableList; +import org.junit.jupiter.api.Test; + +class LogicalJoinSemiJoinTransposeProjectTest implements MemoPatternMatchSupported { + private static final LogicalOlapScan scan1 = PlanConstructor.newLogicalOlapScan(0, "t1", 0); + private static final LogicalOlapScan scan2 = PlanConstructor.newLogicalOlapScan(1, "t2", 0); + private static final LogicalOlapScan scan3 = PlanConstructor.newLogicalOlapScan(2, "t3", 0); + + @Test + public void testSimple() { + LogicalPlan topJoin = new LogicalPlanBuilder(scan1) + .join(scan2, JoinType.LEFT_SEMI_JOIN, Pair.of(0, 0)) // t1.id = t2.id + .project(ImmutableList.of(0, 1)) + .join(scan3, JoinType.INNER_JOIN, Pair.of(0, 0)) // t1.id = t3.id + .build(); + + PlanChecker.from(MemoTestUtils.createConnectContext(), topJoin) + .applyExploration(LogicalJoinSemiJoinTransposeProject.INSTANCE.build()) + .printlnExploration() + .matchesExploration( + leftSemiLogicalJoin( + innerLogicalJoin( + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t1")), + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t3")) + ), + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t2")) + ) + ); + } + + @Test + public void generateTopProject() { + LogicalPlan topJoin = new LogicalPlanBuilder(scan1) + .join(scan2, JoinType.LEFT_SEMI_JOIN, Pair.of(0, 0)) // t1.id = t2.id + .project(ImmutableList.of(0)) + .join(scan3, JoinType.INNER_JOIN, Pair.of(0, 0)) // t1.id = t3.id + .build(); + + PlanChecker.from(MemoTestUtils.createConnectContext(), topJoin) + .applyExploration(LogicalJoinSemiJoinTransposeProject.INSTANCE.build()) + .printlnExploration() + .matchesExploration( + logicalProject( + leftSemiLogicalJoin( + innerLogicalJoin( + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t1")), + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t3")) + ), + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t2")) + ) + ) + ); + } +} diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeTest.java new file mode 100644 index 0000000000..bbc5d6f25f --- /dev/null +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/exploration/join/LogicalJoinSemiJoinTransposeTest.java @@ -0,0 +1,56 @@ +// Licensed to the Apache Software Foundation (ASF) under one +// or more contributor license agreements. See the NOTICE file +// distributed with this work for additional information +// regarding copyright ownership. The ASF licenses this file +// to you under the Apache License, Version 2.0 (the +// "License"); you may not use this file except in compliance +// with the License. You may obtain a copy of the License at +// +// http://www.apache.org/licenses/LICENSE-2.0 +// +// Unless required by applicable law or agreed to in writing, +// software distributed under the License is distributed on an +// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY +// KIND, either express or implied. See the License for the +// specific language governing permissions and limitations +// under the License. + +package org.apache.doris.nereids.rules.exploration.join; + +import org.apache.doris.common.Pair; +import org.apache.doris.nereids.trees.plans.JoinType; +import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; +import org.apache.doris.nereids.trees.plans.logical.LogicalPlan; +import org.apache.doris.nereids.util.LogicalPlanBuilder; +import org.apache.doris.nereids.util.MemoPatternMatchSupported; +import org.apache.doris.nereids.util.MemoTestUtils; +import org.apache.doris.nereids.util.PlanChecker; +import org.apache.doris.nereids.util.PlanConstructor; + +import org.junit.jupiter.api.Test; + +class LogicalJoinSemiJoinTransposeTest implements MemoPatternMatchSupported { + private static final LogicalOlapScan scan1 = PlanConstructor.newLogicalOlapScan(0, "t1", 0); + private static final LogicalOlapScan scan2 = PlanConstructor.newLogicalOlapScan(1, "t2", 0); + private static final LogicalOlapScan scan3 = PlanConstructor.newLogicalOlapScan(2, "t3", 0); + + @Test + public void testSimple() { + LogicalPlan topJoin = new LogicalPlanBuilder(scan1) + .join(scan2, JoinType.LEFT_SEMI_JOIN, Pair.of(0, 0)) // t1.id = t2.id + .join(scan3, JoinType.INNER_JOIN, Pair.of(0, 0)) // t1.id = t3.id + .build(); + + PlanChecker.from(MemoTestUtils.createConnectContext(), topJoin) + .applyExploration(LogicalJoinSemiJoinTranspose.INSTANCE.build()) + .matchesExploration( + leftSemiLogicalJoin( + innerLogicalJoin( + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t1")), + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t3")) + ), + logicalOlapScan().when(scan -> scan.getTable().getName().equals("t2")) + ) + ); + } +} diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/logical/LogicalWindowToPhysicalWindowTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/implementation/LogicalWindowToPhysicalWindowTest.java similarity index 97% rename from fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/logical/LogicalWindowToPhysicalWindowTest.java rename to fe/fe-core/src/test/java/org/apache/doris/nereids/rules/implementation/LogicalWindowToPhysicalWindowTest.java index 0ba1f76ec3..801702b0a6 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/rewrite/logical/LogicalWindowToPhysicalWindowTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/implementation/LogicalWindowToPhysicalWindowTest.java @@ -15,9 +15,8 @@ // specific language governing permissions and limitations // under the License. -package org.apache.doris.nereids.rules.rewrite.logical; +package org.apache.doris.nereids.rules.implementation; -import org.apache.doris.nereids.rules.implementation.LogicalWindowToPhysicalWindow; import org.apache.doris.nereids.rules.implementation.LogicalWindowToPhysicalWindow.WindowFrameGroup; import org.apache.doris.nereids.trees.expressions.Alias; import org.apache.doris.nereids.trees.expressions.Expression; --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org