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/doris.git
The following commit(s) were added to refs/heads/master by this push:
new 83803d1968 [feature](nereids) implementation rules. (#10194)
83803d1968 is described below
commit 83803d1968998a3cba4c0bae7cfd400bcbce2564
Author: jakevin <[email protected]>
AuthorDate: Tue Jun 21 23:34:08 2022 +0800
[feature](nereids) implementation rules. (#10194)
---
.../java/org/apache/doris/nereids/memo/Group.java | 2 +
.../apache/doris/nereids/memo/GroupExpression.java | 18 +++++
.../plans/physical/PhysicalBroadcastHashJoin.java | 77 ----------------------
.../nereids/properties/LogicalProperties.java | 2 +-
.../nereids/properties/PhysicalProperties.java | 1 -
.../org/apache/doris/nereids/rules/RuleSet.java | 4 ++
.../org/apache/doris/nereids/rules/RuleType.java | 5 ++
...oin.java => LogicalFilterToPhysicalFilter.java} | 15 ++---
.../implementation/LogicalJoinToHashJoin.java | 5 +-
... => LogicalProjectionToPhysicalProjection.java} | 15 ++---
.../LogicalProjectionToPhysicalProjectionTest.java | 56 ++++++++++++++++
11 files changed, 102 insertions(+), 98 deletions(-)
diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java
index 52643dc3e8..ad77a424b7 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/Group.java
@@ -41,6 +41,8 @@ public class Group {
private final List<GroupExpression> physicalExpressions =
Lists.newArrayList();
private LogicalProperties logicalProperties;
+ // Map of cost lower bounds
+ // Map required plan props to cost lower bound of corresponding plan
private Map<PhysicalProperties, Pair<Double, GroupExpression>>
lowestCostPlans;
private double costLowerBound = -1;
private boolean isExplored = false;
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/GroupExpression.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/GroupExpression.java
index 6322bf3b4c..19c70fa550 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/GroupExpression.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/memo/GroupExpression.java
@@ -17,14 +17,19 @@
package org.apache.doris.nereids.memo;
+import org.apache.doris.common.Pair;
import org.apache.doris.nereids.operators.Operator;
+import org.apache.doris.nereids.properties.PhysicalProperties;
import org.apache.doris.nereids.rules.Rule;
import org.apache.doris.nereids.rules.RuleType;
import com.clearspring.analytics.util.Lists;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Maps;
import java.util.BitSet;
import java.util.List;
+import java.util.Map;
import java.util.Objects;
/**
@@ -37,6 +42,9 @@ public class GroupExpression {
private final BitSet ruleMasks;
private boolean statDerived;
+ // Mapping from output properties to the corresponding best cost,
statistics, and child properties.
+ private final Map<PhysicalProperties, Pair<Double,
List<PhysicalProperties>>> lowestCostTable;
+
public GroupExpression(Operator op) {
this(op, Lists.newArrayList());
}
@@ -52,6 +60,7 @@ public class GroupExpression {
this.children = Objects.requireNonNull(children);
this.ruleMasks = new BitSet(RuleType.SENTINEL.ordinal());
this.statDerived = false;
+ this.lowestCostTable = Maps.newHashMap();
}
public int arity() {
@@ -106,6 +115,15 @@ public class GroupExpression {
this.statDerived = statDerived;
}
+ public Map<PhysicalProperties, Pair<Double, List<PhysicalProperties>>>
getLowestCostTable() {
+ return lowestCostTable;
+ }
+
+ public List<PhysicalProperties> getInputPropertiesList(PhysicalProperties
require) {
+ Preconditions.checkState(lowestCostTable.containsKey(require));
+ return lowestCostTable.get(require).second;
+ }
+
@Override
public boolean equals(Object o) {
if (this == o) {
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/operators/plans/physical/PhysicalBroadcastHashJoin.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/operators/plans/physical/PhysicalBroadcastHashJoin.java
deleted file mode 100644
index 0a6bfe2ccf..0000000000
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/operators/plans/physical/PhysicalBroadcastHashJoin.java
+++ /dev/null
@@ -1,77 +0,0 @@
-// 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.operators.plans.physical;
-
-import org.apache.doris.nereids.operators.OperatorType;
-import org.apache.doris.nereids.operators.plans.JoinType;
-import org.apache.doris.nereids.trees.expressions.Expression;
-
-import com.google.common.collect.ImmutableList;
-
-import java.util.List;
-import java.util.Objects;
-import java.util.Optional;
-
-/**
- * Physical operator represents broadcast hash join.
- */
-public class PhysicalBroadcastHashJoin extends PhysicalBinaryOperator {
-
- private final JoinType joinType;
- private final Optional<Expression> condition;
-
- /**
- * Constructor for PhysicalBroadcastHashJoin.
- *
- * @param joinType logical join type in Nereids
- */
- public PhysicalBroadcastHashJoin(JoinType joinType) {
- this(joinType, Optional.empty());
- }
-
- /**
- * Constructor for PhysicalBroadcastHashJoin.
- *
- * @param joinType logical join type in Nereids
- * @param condition on clause expression
- */
- public PhysicalBroadcastHashJoin(JoinType joinType, Optional<Expression>
condition) {
- super(OperatorType.PHYSICAL_BROADCAST_HASH_JOIN);
- this.joinType = Objects.requireNonNull(joinType, "joinType can not be
null");
- this.condition = Objects.requireNonNull(condition, "condition can not
be null");
- }
-
- public JoinType getJoinType() {
- return joinType;
- }
-
- public Optional<Expression> getCondition() {
- return condition;
- }
-
- @Override
- public String toString() {
- return "Broadcast Hash Join (" + joinType
- + ", " + condition + ")";
- }
-
- @Override
- public List<Expression> getExpressions() {
- return
condition.<List<Expression>>map(ImmutableList::of).orElseGet(ImmutableList::of);
- }
-}
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/LogicalProperties.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/LogicalProperties.java
index c7fd2b66b3..3623767554 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/LogicalProperties.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/LogicalProperties.java
@@ -47,7 +47,7 @@ public class LogicalProperties {
return outputSupplier.get();
}
- public LogicalProperties withOutput(List<Slot> output) {
+ public static LogicalProperties withOutput(List<Slot> output) {
return new LogicalProperties(Suppliers.ofInstance(output));
}
}
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/PhysicalProperties.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/PhysicalProperties.java
index abe767c40e..88ae78aa87 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/PhysicalProperties.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/properties/PhysicalProperties.java
@@ -21,7 +21,6 @@ package org.apache.doris.nereids.properties;
* Physical properties used in cascades.
*/
public class PhysicalProperties {
-
private DistributionSpec distributionDesc;
public DistributionSpec getDistributionDesc() {
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 4292239b59..657ab82c76 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,7 +20,9 @@ package org.apache.doris.nereids.rules;
import org.apache.doris.nereids.rules.analysis.BindRelation;
import org.apache.doris.nereids.rules.exploration.join.JoinCommutative;
import org.apache.doris.nereids.rules.exploration.join.JoinLeftAssociative;
+import
org.apache.doris.nereids.rules.implementation.LogicalFilterToPhysicalFilter;
import org.apache.doris.nereids.rules.implementation.LogicalJoinToHashJoin;
+import
org.apache.doris.nereids.rules.implementation.LogicalProjectionToPhysicalProjection;
import org.apache.doris.nereids.trees.TreeNode;
import org.apache.doris.nereids.trees.plans.Plan;
@@ -44,6 +46,8 @@ public class RuleSet {
public static final List<Rule<Plan>> IMPLEMENTATION_RULES =
planRuleFactories()
.add(new LogicalJoinToHashJoin())
+ .add(new LogicalProjectionToPhysicalProjection())
+ .add(new LogicalFilterToPhysicalFilter())
.build();
public List<Rule<Plan>> getAnalysisRules() {
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java
index 14fc6eb57b..93f9e846df 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java
@@ -43,6 +43,11 @@ public enum RuleType {
// implementation rules
LOGICAL_JOIN_TO_HASH_JOIN_RULE(RuleTypeClass.IMPLEMENTATION),
+
LOGICAL_PROJECTION_TO_PHYSICAL_PROJECTION_RULE(RuleTypeClass.IMPLEMENTATION),
+ LOGICAL_FILTER_TO_PHYSICAL_FILTER_RULE(RuleTypeClass.IMPLEMENTATION),
+ LOGICAL_LIMIT_TO_PHYSICAL_LIMIT_RULE(RuleTypeClass.IMPLEMENTATION),
+ LOGICAL_JOIN_TO_HASH_AGG_RULE(RuleTypeClass.IMPLEMENTATION),
+ LOGICAL_JOIN_TO_OLAP_SCAN_RULE(RuleTypeClass.IMPLEMENTATION),
IMPLEMENTATION_SENTINEL(RuleTypeClass.IMPLEMENTATION),
// sentinel, use to count rules
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalFilterToPhysicalFilter.java
similarity index 73%
copy from
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
copy to
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalFilterToPhysicalFilter.java
index 2fee5db9a8..db6e95cc98 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalFilterToPhysicalFilter.java
@@ -17,7 +17,7 @@
package org.apache.doris.nereids.rules.implementation;
-import
org.apache.doris.nereids.operators.plans.physical.PhysicalBroadcastHashJoin;
+import org.apache.doris.nereids.operators.plans.physical.PhysicalFilter;
import org.apache.doris.nereids.rules.Rule;
import org.apache.doris.nereids.rules.RuleType;
import org.apache.doris.nereids.trees.plans.Plan;
@@ -25,14 +25,13 @@ import org.apache.doris.nereids.trees.plans.Plan;
/**
* Implementation rule that convert logical join to physical hash join.
*/
-public class LogicalJoinToHashJoin extends OneImplementationRuleFactory {
+public class LogicalFilterToPhysicalFilter extends
OneImplementationRuleFactory {
@Override
public Rule<Plan> build() {
- // fixme, just for example now
- return logicalJoin().then(join -> plan(
- new PhysicalBroadcastHashJoin(join.operator.getJoinType(),
join.operator.getCondition()),
- join.getLogicalProperties(),
- join.left(), join.right()
- )).toRule(RuleType.LOGICAL_JOIN_TO_HASH_JOIN_RULE);
+ return logicalFilter().then(filter -> plan(
+ new PhysicalFilter(filter.getOperator().getPredicates()),
+ filter.getLogicalProperties(),
+ filter.child()
+ )).toRule(RuleType.LOGICAL_FILTER_TO_PHYSICAL_FILTER_RULE);
}
}
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
index 2fee5db9a8..2683e0fcb6 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
@@ -17,7 +17,7 @@
package org.apache.doris.nereids.rules.implementation;
-import
org.apache.doris.nereids.operators.plans.physical.PhysicalBroadcastHashJoin;
+import org.apache.doris.nereids.operators.plans.physical.PhysicalHashJoin;
import org.apache.doris.nereids.rules.Rule;
import org.apache.doris.nereids.rules.RuleType;
import org.apache.doris.nereids.trees.plans.Plan;
@@ -28,9 +28,8 @@ import org.apache.doris.nereids.trees.plans.Plan;
public class LogicalJoinToHashJoin extends OneImplementationRuleFactory {
@Override
public Rule<Plan> build() {
- // fixme, just for example now
return logicalJoin().then(join -> plan(
- new PhysicalBroadcastHashJoin(join.operator.getJoinType(),
join.operator.getCondition()),
+ new PhysicalHashJoin(join.operator.getJoinType(),
join.operator.getCondition().get()),
join.getLogicalProperties(),
join.left(), join.right()
)).toRule(RuleType.LOGICAL_JOIN_TO_HASH_JOIN_RULE);
diff --git
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalProjectionToPhysicalProjection.java
similarity index 73%
copy from
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
copy to
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalProjectionToPhysicalProjection.java
index 2fee5db9a8..66a2b8ad73 100644
---
a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalJoinToHashJoin.java
+++
b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/implementation/LogicalProjectionToPhysicalProjection.java
@@ -17,7 +17,7 @@
package org.apache.doris.nereids.rules.implementation;
-import
org.apache.doris.nereids.operators.plans.physical.PhysicalBroadcastHashJoin;
+import org.apache.doris.nereids.operators.plans.physical.PhysicalProject;
import org.apache.doris.nereids.rules.Rule;
import org.apache.doris.nereids.rules.RuleType;
import org.apache.doris.nereids.trees.plans.Plan;
@@ -25,14 +25,13 @@ import org.apache.doris.nereids.trees.plans.Plan;
/**
* Implementation rule that convert logical join to physical hash join.
*/
-public class LogicalJoinToHashJoin extends OneImplementationRuleFactory {
+public class LogicalProjectionToPhysicalProjection extends
OneImplementationRuleFactory {
@Override
public Rule<Plan> build() {
- // fixme, just for example now
- return logicalJoin().then(join -> plan(
- new PhysicalBroadcastHashJoin(join.operator.getJoinType(),
join.operator.getCondition()),
- join.getLogicalProperties(),
- join.left(), join.right()
- )).toRule(RuleType.LOGICAL_JOIN_TO_HASH_JOIN_RULE);
+ return logicalProject().then(projection -> plan(
+ new PhysicalProject(projection.getOperator().getProjects()),
+ projection.getLogicalProperties(),
+ projection.child()
+ )).toRule(RuleType.LOGICAL_PROJECTION_TO_PHYSICAL_PROJECTION_RULE);
}
}
diff --git
a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/implementation/LogicalProjectionToPhysicalProjectionTest.java
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/implementation/LogicalProjectionToPhysicalProjectionTest.java
new file mode 100644
index 0000000000..84df7938dd
--- /dev/null
+++
b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/implementation/LogicalProjectionToPhysicalProjectionTest.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.implementation;
+
+import org.apache.doris.nereids.OptimizerContext;
+import org.apache.doris.nereids.PlannerContext;
+import org.apache.doris.nereids.memo.Group;
+import org.apache.doris.nereids.memo.Memo;
+import org.apache.doris.nereids.operators.OperatorType;
+import org.apache.doris.nereids.operators.plans.logical.LogicalProject;
+import org.apache.doris.nereids.properties.PhysicalProperties;
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.trees.plans.GroupPlan;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.trees.plans.Plans;
+import org.apache.doris.qe.ConnectContext;
+
+import com.google.common.collect.Lists;
+import mockit.Mocked;
+import org.junit.Assert;
+import org.junit.Test;
+
+import java.util.List;
+
+public class LogicalProjectionToPhysicalProjectionTest implements Plans {
+ @Test
+ public void projectionImplTest(@Mocked Group group) {
+ LogicalProject logicalProject = new
LogicalProject(Lists.newArrayList());
+ Plan plan = plan(logicalProject, new GroupPlan(group));
+
+ Rule<Plan> rule = new LogicalProjectionToPhysicalProjection().build();
+
+ PlannerContext plannerContext = new PlannerContext(new
OptimizerContext(new Memo()), new ConnectContext(),
+ new PhysicalProperties());
+ List<Plan> transform = rule.transform(plan, plannerContext);
+ Assert.assertEquals(1, transform.size());
+
+ Plan implPlan = transform.get(0);
+ Assert.assertEquals(OperatorType.PHYSICAL_PROJECT,
implPlan.getOperator().getType());
+ }
+}
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]