You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by GitBox <gi...@apache.org> on 2022/06/25 15:37:45 UTC

[GitHub] [doris] morrySnow commented on a diff in pull request #10412: [Feature] [Draft] Agg rewrite rule of nereids optmizer

morrySnow commented on code in PR #10412:
URL: https://github.com/apache/doris/pull/10412#discussion_r906694231


##########
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleSet.java:
##########
@@ -37,6 +38,7 @@
 public class RuleSet {
     public static final List<Rule<Plan>> ANALYSIS_RULES = planRuleFactories()
             .add(new BindRelation())
+            .add(new AggregateRewrite())

Review Comment:
   maybe it is better to use two rule set, one for bind, another for rewrite



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/AggregateRewrite.java:
##########
@@ -0,0 +1,93 @@
+// 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.rewrite;
+
+import org.apache.doris.analysis.FunctionName;
+import org.apache.doris.catalog.AggregateFunction;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.Function;
+import org.apache.doris.catalog.Function.CompareMode;
+import org.apache.doris.catalog.Type;
+import org.apache.doris.nereids.operators.Operator;
+import org.apache.doris.nereids.operators.plans.logical.LogicalAggregation;
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.rules.RuleType;
+import org.apache.doris.nereids.trees.analysis.FunctionParams;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.FunctionCall;
+import org.apache.doris.nereids.trees.expressions.NamedExpression;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.types.DataType;
+
+import com.google.common.base.Preconditions;
+
+import java.util.List;
+import java.util.stream.Collectors;
+
+/**
+ * Used to generate the merge agg node for distributed execution.
+ */
+public class AggregateRewrite extends OneRewriteRuleFactory {
+
+    @Override
+    public Rule<Plan> build() {
+        return logicalAggregation().thenApply(ctx -> {

Review Comment:
   we need to check whether this aggregate plan has been split or generated by this rule. If that, we should not apply this rule again.



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/AggregateRewrite.java:
##########
@@ -0,0 +1,93 @@
+// 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.rewrite;
+
+import org.apache.doris.analysis.FunctionName;
+import org.apache.doris.catalog.AggregateFunction;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.Function;
+import org.apache.doris.catalog.Function.CompareMode;
+import org.apache.doris.catalog.Type;
+import org.apache.doris.nereids.operators.Operator;
+import org.apache.doris.nereids.operators.plans.logical.LogicalAggregation;
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.rules.RuleType;
+import org.apache.doris.nereids.trees.analysis.FunctionParams;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.FunctionCall;
+import org.apache.doris.nereids.trees.expressions.NamedExpression;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.types.DataType;
+
+import com.google.common.base.Preconditions;
+
+import java.util.List;
+import java.util.stream.Collectors;
+
+/**
+ * Used to generate the merge agg node for distributed execution.
+ */
+public class AggregateRewrite extends OneRewriteRuleFactory {
+
+    @Override
+    public Rule<Plan> build() {
+        return logicalAggregation().thenApply(ctx -> {
+            Plan plan = ctx.root;
+            Operator operator = plan.getOperator();
+            LogicalAggregation agg = (LogicalAggregation) operator;
+            List<NamedExpression> namedExpressionList = agg.getOutputExpressions();
+            List<NamedExpression> intermediateAggExpressionList = agg.getOutputExpressions();
+            for (NamedExpression namedExpression : namedExpressionList) {

Review Comment:
   the slot reference in upper node's expression always refer to it children's output. So we should not only update function call's type, but also update all slot references.



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/operators/plans/logical/LogicalAggregation.java:
##########
@@ -42,18 +42,27 @@
 public class LogicalAggregation extends LogicalUnaryOperator {
 
     private final List<Expression> groupByExpressions;
-    private final List<? extends NamedExpression> outputExpressions;
+    private List<NamedExpression> outputExpressions;

Review Comment:
   all operator should immutable, always generate new operator if u want to change attributes in it



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/RuleType.java:
##########
@@ -34,6 +34,7 @@ public enum RuleType {
     // rewrite rules
     COLUMN_PRUNE_PROJECTION(RuleTypeClass.REWRITE),
     REWRITE_SENTINEL(RuleTypeClass.REWRITE),
+    REWRITE_AGG(RuleTypeClass.REWRITE),

Review Comment:
   should be move above REWRITE_SENTINEL



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/AggregateRewrite.java:
##########
@@ -0,0 +1,93 @@
+// 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.rewrite;
+
+import org.apache.doris.analysis.FunctionName;
+import org.apache.doris.catalog.AggregateFunction;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.Function;
+import org.apache.doris.catalog.Function.CompareMode;
+import org.apache.doris.catalog.Type;
+import org.apache.doris.nereids.operators.Operator;
+import org.apache.doris.nereids.operators.plans.logical.LogicalAggregation;
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.rules.RuleType;
+import org.apache.doris.nereids.trees.analysis.FunctionParams;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.FunctionCall;
+import org.apache.doris.nereids.trees.expressions.NamedExpression;
+import org.apache.doris.nereids.trees.plans.Plan;
+import org.apache.doris.nereids.types.DataType;
+
+import com.google.common.base.Preconditions;
+
+import java.util.List;
+import java.util.stream.Collectors;
+
+/**
+ * Used to generate the merge agg node for distributed execution.
+ */
+public class AggregateRewrite extends OneRewriteRuleFactory {
+
+    @Override
+    public Rule<Plan> build() {
+        return logicalAggregation().thenApply(ctx -> {
+            Plan plan = ctx.root;
+            Operator operator = plan.getOperator();
+            LogicalAggregation agg = (LogicalAggregation) operator;
+            List<NamedExpression> namedExpressionList = agg.getOutputExpressions();

Review Comment:
   ```suggestion
               List<NamedExpression> outputExpressionList = agg.getOutputExpressions();
   ```



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/FunctionCall.java:
##########
@@ -30,7 +35,14 @@ public class FunctionCall extends Expression {
 
     private FunctionParams fnParams;
 
-    private FunctionCall(FunctionName functionName, FunctionParams functionParams) {
+    private DataType retType;
+
+    // Used to construct output, this type may differ from above retType
+    // when the intermediate type of aggregate function is not same
+    // as its return type
+    private DataType type;

Review Comment:
   do not understand why introduce these two type



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


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