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


##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java:
##########
@@ -255,12 +255,12 @@ public PlanFragment 
visitPhysicalOlapScan(PhysicalOlapScan olapScan, PlanTransla
             throw new AnalysisException(e.getMessage());
         }
         Utils.execWithUncheckedException(olapScanNode::init);
-        olapScanNode.addConjuncts(execConjunctsList);
         context.addScanNode(olapScanNode);
         // Create PlanFragment
         // TODO: add data partition after we have physical properties
         PlanFragment planFragment = new PlanFragment(context.nextFragmentId(), 
olapScanNode, DataPartition.RANDOM);
         context.addPlanFragment(planFragment);
+        olapScanNode.setIsPreAggregation(true, "Nereids support duplicate 
table only for now");

Review Comment:
   add a TODO pls



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java:
##########
@@ -125,10 +126,12 @@ public PlanFragment translatePlan(PhysicalPlan 
physicalPlan, PlanTranslatorConte
         // TODO: trick here, we need push project down
         if (physicalPlan.getType() == PlanType.PHYSICAL_PROJECT) {
             PhysicalProject<Plan> physicalProject = (PhysicalProject<Plan>) 
physicalPlan;
-            List<Expr> outputExprs = physicalProject.getProjects().stream()
-                    .map(e -> ExpressionTranslator.translate(e, context))
-                    .collect(Collectors.toList());
-            rootFragment.setOutputExprs(outputExprs);
+            rootFragment.setOutputExprs(
+                    physicalProject
+                            .getOutput()
+                            .stream()
+                            .map(s -> ExpressionTranslator.translate(s, 
context))
+                            .collect(Collectors.toList()));

Review Comment:
   why not put projection info on the root PlanNode and remove this branch



##########
fe/fe-core/src/main/java/org/apache/doris/analysis/SlotDescriptor.java:
##########
@@ -338,6 +338,7 @@ public String getExplainString(String prefix) {
         
builder.append(prefix).append("byteOffset=").append(byteOffset).append("\n");
         
builder.append(prefix).append("nullIndicatorByte=").append(nullIndicatorByte).append("\n");
         
builder.append(prefix).append("nullIndicatorBit=").append(nullIndicatorBit).append("\n");
+        
builder.append(prefix).append("nullable=").append(isNullable).append("\n");

Review Comment:
   i don't think this is a good idea to add this in explain string since BE do 
not use this attribute anymore. It could mislead users if it show different 
nullable info with nullIndicatorByte and nullIndicatorBit



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/ExpressionTranslator.java:
##########
@@ -258,9 +259,11 @@ public Expr visitBoundFunction(BoundFunction function, 
PlanTranslatorContext con
 
     @Override
     public Expr visitBinaryArithmetic(BinaryArithmetic binaryArithmetic, 
PlanTranslatorContext context) {
-        return new ArithmeticExpr(binaryArithmetic.getLegacyOperator(),
+        ArithmeticExpr arithmeticExpr =  new 
ArithmeticExpr(binaryArithmetic.getLegacyOperator(),
                 binaryArithmetic.child(0).accept(this, context),
                 binaryArithmetic.child(1).accept(this, context));
+        Utils.execWithUncheckedException(() -> 
arithmeticExpr.analyzeImpl(null));

Review Comment:
   we have do it in `finalizeImplForNereids`, if it has any bugs, please fix it 
and remove this line



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java:
##########
@@ -434,28 +432,65 @@ public PlanFragment 
visitPhysicalProject(PhysicalProject<Plan> project, PlanTran
                 .map(e -> ExpressionTranslator.translate(e, context))
                 .collect(Collectors.toList());
         // TODO: fix the project alias of an aliased relation.
+        List<Slot> slotList = project.getOutput();
+        TupleDescriptor tupleDescriptor = generateTupleDesc(slotList, null, 
context);
         PlanNode inputPlanNode = inputFragment.getPlanRoot();
+        if (inputPlanNode instanceof HashJoinNode) {
+            HashJoinNode hashJoinNode = (HashJoinNode) inputPlanNode;
+            hashJoinNode.setvOutputTupleDesc(tupleDescriptor);
+            hashJoinNode.setvSrcToOutputSMap(execExprList);
+            return inputFragment;
+        }
+        inputPlanNode.setProjectList(execExprList);
+        inputPlanNode.setOutputTupleDesc(tupleDescriptor);
+
         List<Expr> predicateList = inputPlanNode.getConjuncts();
         Set<Integer> requiredSlotIdList = new HashSet<>();
         for (Expr expr : predicateList) {
             extractExecSlot(expr, requiredSlotIdList);
         }
         for (Expr expr : execExprList) {
-            if (expr instanceof SlotRef) {
-                requiredSlotIdList.add(((SlotRef) 
expr).getDesc().getId().asInt());
-            }
+            extractExecSlot(expr, requiredSlotIdList);
+        }
+        if (inputPlanNode instanceof OlapScanNode) {
+            updateChildSlotsMaterialization(inputPlanNode, requiredSlotIdList, 
context);
         }
         return inputFragment;
     }
 
+    private void updateChildSlotsMaterialization(PlanNode execPlan,
+            Set<Integer> requiredSlotIdList,
+            PlanTranslatorContext context) {
+        Set<SlotRef> slotRefSet = new HashSet<>();
+        for (Expr expr : execPlan.getConjuncts()) {
+            expr.collect(SlotRef.class, slotRefSet);
+        }
+        Set<Integer> slotIdSet = slotRefSet.stream()
+                
.map(SlotRef::getSlotId).map(SlotId::asInt).collect(Collectors.toSet());
+        slotIdSet.addAll(requiredSlotIdList);
+        execPlan.getTupleIds().stream()
+                .map(context::getTupleDesc)
+                .map(TupleDescriptor::getSlots)
+                .flatMap(List::stream)
+                .forEach(s -> {
+                    s.setIsMaterialized(slotIdSet.contains(s.getId().asInt()));

Review Comment:
   i think lambda expression is ok, lambda statement is not necessary



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PlanTranslatorContext.java:
##########
@@ -110,23 +113,25 @@ public SlotDescriptor createSlotDesc(TupleDescriptor 
tupleDesc, SlotReference sl
         }
         
slotDescriptor.setType(slotReference.getDataType().toCatalogDataType());
         slotDescriptor.setIsMaterialized(true);
+        slotDescriptor.setIsNullable(slotReference.nullable());

Review Comment:
   👍🏻



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/jobs/batch/RewriteJob.java:
##########
@@ -51,6 +53,10 @@ public RewriteJob(CascadesContext cascadesContext) {
                 .add(topDownBatch(ImmutableList.of(new 
FindHashConditionForJoin())))
                 .add(topDownBatch(ImmutableList.of(new 
PushPredicateThroughJoin())))
                 .add(topDownBatch(ImmutableList.of(new 
AggregateDisassemble())))
+                .add(topDownBatch(ImmutableList.of(new ColumnPruning())))
+                .add(topDownBatch(ImmutableList.of(new 
SwapFilterAndProject())))
+                .add(topDownBatch(ImmutableList.of(new 
MergeConsecutiveProjects())))
+                .add(topDownBatch(ImmutableList.of(new 
MergeConsecutiveFilters())))

Review Comment:
   i think we need to add these three rules in one job



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PlanTranslatorContext.java:
##########
@@ -110,23 +113,25 @@ public SlotDescriptor createSlotDesc(TupleDescriptor 
tupleDesc, SlotReference sl
         }
         
slotDescriptor.setType(slotReference.getDataType().toCatalogDataType());
         slotDescriptor.setIsMaterialized(true);
+        slotDescriptor.setIsNullable(slotReference.nullable());
         this.addExprIdPair(slotReference.getExprId(), new 
SlotRef(slotDescriptor));
         return slotDescriptor;
     }
 
-    /**
-     * Create slotDesc with Expression.
-     */
-    public void createSlotDesc(TupleDescriptor tupleDesc, Expression 
expression) {
-        SlotDescriptor slotDescriptor = this.addSlotDesc(tupleDesc);
-        slotDescriptor.setType(expression.getDataType().toCatalogDataType());
-    }
-
     public TupleDescriptor getTupleDesc(TupleId tupleId) {
         return descTable.getTupleDesc(tupleId);
     }
 
     public DescriptorTable getDescTable() {
         return descTable;
     }
+
+    public void putPlanToExprIdMapping(PhysicalPlan plan, Set<ExprId> exprId) {
+        requiredSlotOfEachPhysicalOperator.put(plan, exprId);
+    }
+
+    public Set<ExprId> getRequiredExprId(PhysicalPlan plan) {
+        return requiredSlotOfEachPhysicalOperator.get(plan);
+    }

Review Comment:
   useless code



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/Expression.java:
##########
@@ -146,4 +153,34 @@ public boolean equals(Object o) {
     public int hashCode() {
         return 0;
     }
+
+    public List<DataType> getChildrenType() {
+        List<DataType> typeList = new ArrayList<>();
+        for (Expression child : children) {
+            typeList.add(child.getDataType());
+        }
+        return typeList;
+    }
+
+    /**
+     * Collect CatalogType of children.
+     * @return Catalog type array
+     */
+    public Type[] getChildrenCatalogType() {
+        int childrenSize = children.size();
+        Type[] typeList = new Type[childrenSize];
+        for (int i = 0; i < childrenSize; i++) {
+            typeList[i] = child(i).getDataType().toCatalogDataType();
+        }
+        return typeList;
+    }
+
+    protected Function getBuiltinFunction(String name, Type[] argTypes, 
Function.CompareMode mode)  {
+        FunctionName fnName = new FunctionName(name);
+        Function searchDesc = new Function(fnName, Arrays.asList(argTypes), 
Type.INVALID, false,
+                VectorizedUtil.isVectorized());
+        // TODO: process RAND function.
+        return Env.getCurrentEnv().getFunction(searchDesc, mode);
+    }
+

Review Comment:
   remove it



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PlanTranslatorContext.java:
##########
@@ -54,6 +55,8 @@ public class PlanTranslatorContext {
 
     private final List<ScanNode> scanNodeList = new ArrayList<>();
 
+    private final Map<PhysicalPlan, Set<ExprId>> 
requiredSlotOfEachPhysicalOperator = new HashMap<>();
+

Review Comment:
   remove it



##########
fe/fe-core/src/main/java/org/apache/doris/planner/HashJoinNode.java:
##########
@@ -1080,13 +1080,18 @@ protected void toThrift(TPlanNode msg) {
         }
         if (vSrcToOutputSMap != null) {
             for (int i = 0; i < vSrcToOutputSMap.size(); i++) {
+                // Enable it after we support new optimizers
+                // if 
(ConnectContext.get().getSessionVariable().isEnableNereidsPlanner()) {
+                //     
msg.addToProjections(vSrcToOutputSMap.getLhs().get(i).treeToThrift());
+                // } else
                 
msg.hash_join_node.addToSrcExprList(vSrcToOutputSMap.getLhs().get(i).treeToThrift());
-                
msg.addToProjections(vSrcToOutputSMap.getLhs().get(i).treeToThrift());
             }
         }
+        // msg.setOutputTupleId(vOutputTupleDesc.getId().asInt());

Review Comment:
   ```suggestion
           // TODO Enable it after we support new optimizers
           // msg.setOutputTupleId(vOutputTupleDesc.getId().asInt());
   ```



##########
fe/fe-core/src/main/java/org/apache/doris/planner/PlanNode.java:
##########
@@ -142,6 +139,10 @@ public abstract class PlanNode extends TreeNode<PlanNode> 
implements PlanStats {
     protected StatisticalType statisticalType = StatisticalType.DEFAULT;
     protected StatsDeriveResult statsDeriveResult;
 
+    protected TupleDescriptor outputTupleDesc;
+
+    protected List<Expr> projectList;

Review Comment:
   ```suggestion
       protected List<Expr> projections;
   ```



##########
fe/fe-core/src/main/java/org/apache/doris/planner/HashJoinNode.java:
##########
@@ -1080,13 +1080,18 @@ protected void toThrift(TPlanNode msg) {
         }
         if (vSrcToOutputSMap != null) {
             for (int i = 0; i < vSrcToOutputSMap.size(); i++) {
+                // Enable it after we support new optimizers
+                // if 
(ConnectContext.get().getSessionVariable().isEnableNereidsPlanner()) {
+                //     
msg.addToProjections(vSrcToOutputSMap.getLhs().get(i).treeToThrift());
+                // } else
                 
msg.hash_join_node.addToSrcExprList(vSrcToOutputSMap.getLhs().get(i).treeToThrift());
-                
msg.addToProjections(vSrcToOutputSMap.getLhs().get(i).treeToThrift());
             }
         }
+        // msg.setOutputTupleId(vOutputTupleDesc.getId().asInt());
         if (vOutputTupleDesc != null) {
             
msg.hash_join_node.setVoutputTupleId(vOutputTupleDesc.getId().asInt());
-            msg.setOutputTupleId(vOutputTupleDesc.getId().asInt());
+            // Enable it after we support new optimizers

Review Comment:
   ```suggestion
               // TODO: Enable it after we support new optimizers
   ```



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java:
##########
@@ -434,28 +432,65 @@ public PlanFragment 
visitPhysicalProject(PhysicalProject<Plan> project, PlanTran
                 .map(e -> ExpressionTranslator.translate(e, context))
                 .collect(Collectors.toList());
         // TODO: fix the project alias of an aliased relation.
+        List<Slot> slotList = project.getOutput();

Review Comment:
   To a common format, we do not use List, Set, ... as suffix usually, instead 
we use plural, i.e. slots. Please fix other place in this file, thanks



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/glue/translator/PhysicalPlanTranslator.java:
##########
@@ -434,28 +432,65 @@ public PlanFragment 
visitPhysicalProject(PhysicalProject<Plan> project, PlanTran
                 .map(e -> ExpressionTranslator.translate(e, context))
                 .collect(Collectors.toList());
         // TODO: fix the project alias of an aliased relation.
+        List<Slot> slotList = project.getOutput();
+        TupleDescriptor tupleDescriptor = generateTupleDesc(slotList, null, 
context);
         PlanNode inputPlanNode = inputFragment.getPlanRoot();
+        if (inputPlanNode instanceof HashJoinNode) {

Review Comment:
   add a comment to explain why HashJoinNode is special and add a TODO that 
remove it when BE fix this special case



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/visitor/RewriteAliasToChildExpr.java:
##########
@@ -0,0 +1,37 @@
+// 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.trees.expressions.visitor;
+
+import org.apache.doris.nereids.trees.expressions.Alias;
+import org.apache.doris.nereids.trees.expressions.Expression;
+import org.apache.doris.nereids.trees.expressions.Slot;
+import org.apache.doris.nereids.trees.expressions.SlotReference;
+
+import java.util.Map;
+
+/**
+ * UpdateAliasVisitor
+ */
+public class RewriteAliasToChildExpr extends 
DefaultExpressionRewriter<Map<Slot, Alias>> {

Review Comment:
   we do not need this rule, if we replace expression with alias child directly



##########
fe/fe-core/src/main/java/org/apache/doris/planner/ExchangeNode.java:
##########
@@ -83,6 +84,11 @@ public ExchangeNode(PlanNodeId id, PlanNode inputNode, 
boolean copyConjuncts) {
             limit = inputNode.limit;
         }
         computeTupleIds();
+        TupleDescriptor outputTupleDesc = inputNode.getOutputTupleDesc();
+        if (outputTupleDesc != null) {
+            tupleIds.clear();
+            tupleIds.add(outputTupleDesc.getId());
+        }

Review Comment:
   put it into computeTupleIds()



##########
fe/fe-core/src/main/java/org/apache/doris/planner/HashJoinNode.java:
##########
@@ -1080,13 +1080,18 @@ protected void toThrift(TPlanNode msg) {
         }
         if (vSrcToOutputSMap != null) {
             for (int i = 0; i < vSrcToOutputSMap.size(); i++) {
+                // Enable it after we support new optimizers

Review Comment:
   ```suggestion
                   // TODO Enable it after we support new optimizers
   ```



##########
fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/physical/AbstractPhysicalPlan.java:
##########
@@ -64,6 +64,11 @@ public List<Slot> getOutput() {
         return logicalProperties.getOutput();
     }
 
+    @Override

Review Comment:
   move it to `AbstractPlan`, BTW, move getOutput to `AbstractPlan` 



-- 
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-unsubscr...@doris.apache.org

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


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

Reply via email to