You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by mo...@apache.org on 2023/04/21 06:28:15 UTC

[doris] branch master updated: [fix](nereids) LogicalProject should always has non-empty project list (#18863)

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

morrysnow 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 c41b486e7e [fix](nereids) LogicalProject should always has non-empty project list (#18863)
c41b486e7e is described below

commit c41b486e7e38449d0772d4111a78637ab8a1f245
Author: starocean999 <40...@users.noreply.github.com>
AuthorDate: Fri Apr 21 14:28:07 2023 +0800

    [fix](nereids) LogicalProject should always has non-empty project list (#18863)
---
 .../rules/rewrite/logical/ColumnPruning.java       |  3 -
 .../trees/plans/logical/LogicalProject.java        | 11 ++-
 .../apache/doris/nereids/util/ExpressionUtils.java |  3 +
 .../pattern/GroupExpressionMatchingTest.java       | 14 +++-
 .../data/nereids_syntax_p0/column_prune.out        |  4 +
 .../suites/nereids_syntax_p0/column_prune.groovy   | 89 ++++++++++++++++++++++
 6 files changed, 117 insertions(+), 7 deletions(-)

diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/ColumnPruning.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/ColumnPruning.java
index 8b79e73470..63cea0ff4c 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/ColumnPruning.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/rewrite/logical/ColumnPruning.java
@@ -251,9 +251,6 @@ public class ColumnPruning extends DefaultPlanRewriter<PruneContext> implements
         for (Plan child : plan.children()) {
             Set<Slot> childOutputSet = child.getOutputSet();
             Set<Slot> childRequiredSlots = Sets.intersection(childrenRequiredSlots, childOutputSet);
-            if (childRequiredSlots.isEmpty()) {
-                childRequiredSlots = ImmutableSet.of(ExpressionUtils.selectMinimumColumn(childOutputSet));
-            }
             Plan prunedChild = doPruneChild(plan, child, childRequiredSlots);
             if (prunedChild != child) {
                 hasNewChildren = true;
diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java
index c13ffa518c..418471c851 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalProject.java
@@ -17,6 +17,7 @@
 
 package org.apache.doris.nereids.trees.plans.logical;
 
+import org.apache.doris.nereids.analyzer.Unbound;
 import org.apache.doris.nereids.analyzer.UnboundStar;
 import org.apache.doris.nereids.memo.GroupExpression;
 import org.apache.doris.nereids.properties.LogicalProperties;
@@ -28,6 +29,7 @@ import org.apache.doris.nereids.trees.plans.Plan;
 import org.apache.doris.nereids.trees.plans.PlanType;
 import org.apache.doris.nereids.trees.plans.algebra.Project;
 import org.apache.doris.nereids.trees.plans.visitor.PlanVisitor;
+import org.apache.doris.nereids.util.ExpressionUtils;
 import org.apache.doris.nereids.util.Utils;
 
 import com.google.common.base.Preconditions;
@@ -82,7 +84,14 @@ public class LogicalProject<CHILD_TYPE extends Plan> extends LogicalUnary<CHILD_
             Optional<GroupExpression> groupExpression, Optional<LogicalProperties> logicalProperties,
             CHILD_TYPE child, boolean isDistinct) {
         super(PlanType.LOGICAL_PROJECT, groupExpression, logicalProperties, child);
-        this.projects = ImmutableList.copyOf(Objects.requireNonNull(projects, "projects can not be null"));
+        Preconditions.checkArgument(projects != null, "projects can not be null");
+        // only ColumnPrune rule may produce empty projects, this happens in rewrite phase
+        // so if projects is empty, all plans have been bound already.
+        Preconditions.checkArgument(!projects.isEmpty() || !(child instanceof Unbound),
+                "projects can not be empty when child plan is unbound");
+        this.projects = projects.isEmpty()
+                ? ImmutableList.of(ExpressionUtils.selectMinimumColumn(child.getOutput()))
+                : projects;
         this.excepts = ImmutableList.copyOf(excepts);
         this.canEliminate = canEliminate;
         this.isDistinct = isDistinct;
diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
index e7b068dc11..606f13f4ca 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/util/ExpressionUtils.java
@@ -212,6 +212,9 @@ public class ExpressionUtils {
                 minSlot = slot;
             } else {
                 int slotDataTypeWidth = slot.getDataType().width();
+                if (slotDataTypeWidth < 0) {
+                    continue;
+                }
                 minSlot = minSlot.getDataType().width() > slotDataTypeWidth ? slot : minSlot;
             }
         }
diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/pattern/GroupExpressionMatchingTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/pattern/GroupExpressionMatchingTest.java
index 4947570ace..78f4df57a3 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/nereids/pattern/GroupExpressionMatchingTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/pattern/GroupExpressionMatchingTest.java
@@ -22,6 +22,7 @@ import org.apache.doris.nereids.analyzer.UnboundSlot;
 import org.apache.doris.nereids.memo.Memo;
 import org.apache.doris.nereids.rules.RulePromise;
 import org.apache.doris.nereids.trees.expressions.EqualTo;
+import org.apache.doris.nereids.trees.expressions.SlotReference;
 import org.apache.doris.nereids.trees.plans.GroupPlan;
 import org.apache.doris.nereids.trees.plans.JoinType;
 import org.apache.doris.nereids.trees.plans.Plan;
@@ -31,6 +32,7 @@ import org.apache.doris.nereids.trees.plans.logical.LogicalJoin;
 import org.apache.doris.nereids.trees.plans.logical.LogicalProject;
 import org.apache.doris.nereids.trees.plans.logical.RelationUtil;
 import org.apache.doris.nereids.trees.plans.visitor.PlanVisitor;
+import org.apache.doris.nereids.types.StringType;
 
 import com.google.common.collect.ImmutableList;
 import com.google.common.collect.ImmutableSet;
@@ -64,7 +66,9 @@ public class GroupExpressionMatchingTest {
                 new Pattern<>(PlanType.LOGICAL_UNBOUND_RELATION));
 
         Plan leaf = new UnboundRelation(RelationUtil.newRelationId(), Lists.newArrayList("test"));
-        LogicalProject root = new LogicalProject(Lists.newArrayList(), leaf);
+        LogicalProject root = new LogicalProject(ImmutableList
+                .of(new SlotReference("name", StringType.INSTANCE, true, ImmutableList.of("test"))),
+                leaf);
         Memo memo = new Memo(root);
 
         Plan anotherLeaf = new UnboundRelation(RelationUtil.newRelationId(), Lists.newArrayList("test2"));
@@ -93,7 +97,9 @@ public class GroupExpressionMatchingTest {
         Pattern pattern = new Pattern<>(PlanType.LOGICAL_PROJECT, Pattern.GROUP);
 
         Plan leaf = new UnboundRelation(RelationUtil.newRelationId(), Lists.newArrayList("test"));
-        LogicalProject root = new LogicalProject(Lists.newArrayList(), leaf);
+        LogicalProject root = new LogicalProject(ImmutableList
+                .of(new SlotReference("name", StringType.INSTANCE, true, ImmutableList.of("test"))),
+                leaf);
         Memo memo = new Memo(root);
 
         Plan anotherLeaf = new UnboundRelation(RelationUtil.newRelationId(), Lists.newArrayList("test2"));
@@ -130,7 +136,9 @@ public class GroupExpressionMatchingTest {
 
     @Test
     public void testAnyWithChild() {
-        Plan root = new LogicalProject(Lists.newArrayList(),
+        Plan root = new LogicalProject(
+                ImmutableList.of(new SlotReference("name", StringType.INSTANCE, true,
+                        ImmutableList.of("test"))),
                 new UnboundRelation(RelationUtil.newRelationId(), Lists.newArrayList("test")));
         Memo memo = new Memo(root);
 
diff --git a/regression-test/data/nereids_syntax_p0/column_prune.out b/regression-test/data/nereids_syntax_p0/column_prune.out
new file mode 100644
index 0000000000..72d126351a
--- /dev/null
+++ b/regression-test/data/nereids_syntax_p0/column_prune.out
@@ -0,0 +1,4 @@
+-- This file is automatically generated. You should know what you did if you want to edit this
+-- !select --
+1
+
diff --git a/regression-test/suites/nereids_syntax_p0/column_prune.groovy b/regression-test/suites/nereids_syntax_p0/column_prune.groovy
new file mode 100644
index 0000000000..dba06b3578
--- /dev/null
+++ b/regression-test/suites/nereids_syntax_p0/column_prune.groovy
@@ -0,0 +1,89 @@
+// 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.
+
+suite("column_prune") {
+    sql "SET enable_nereids_planner=true"
+    sql "SET enable_fallback_to_original_planner=false"
+
+    sql """ DROP TABLE IF EXISTS `test_prune1` """
+    sql """ DROP TABLE IF EXISTS `test_prune2` """
+    sql """ DROP TABLE IF EXISTS `test_prune3` """
+    sql """
+        CREATE TABLE `test_prune1` (
+        `id` varchar(64) NULL,
+        `name` varchar(64) NULL,
+        `age` int NULL
+        ) ENGINE=OLAP
+        DUPLICATE KEY(`id`,`name`)
+        COMMENT 'OLAP'
+        DISTRIBUTED BY HASH(`id`,`name`) BUCKETS 4
+        PROPERTIES (
+        "replication_allocation" = "tag.location.default: 1",
+        "in_memory" = "false",
+        "storage_format" = "V2",
+        "disable_auto_compaction" = "false"
+        );
+    """
+    sql """
+        CREATE TABLE `test_prune2` (
+        `id` varchar(64) NULL,
+        `name` varchar(64) NULL,
+        `age` int NULL
+        ) ENGINE=OLAP
+        DUPLICATE KEY(`id`,`name`)
+        COMMENT 'OLAP'
+        DISTRIBUTED BY HASH(`id`,`name`) BUCKETS 5
+        PROPERTIES (
+        "replication_allocation" = "tag.location.default: 1",
+        "in_memory" = "false",
+        "storage_format" = "V2",
+        "disable_auto_compaction" = "false"
+        );
+    """
+
+    sql """
+        CREATE TABLE `test_prune3` (
+        `id` varchar(64) NULL,
+        `name` varchar(64) NULL,
+        `age` int NULL
+        ) ENGINE=OLAP
+        DUPLICATE KEY(`id`,`name`)
+        COMMENT 'OLAP'
+        DISTRIBUTED BY HASH(`id`,`name`) BUCKETS 6
+        PROPERTIES (
+        "replication_allocation" = "tag.location.default: 1",
+        "in_memory" = "false",
+        "storage_format" = "V2",
+        "disable_auto_compaction" = "false"
+        );
+    """
+
+    sql """insert into test_prune1 values('1','a',12);"""
+    sql """insert into test_prune2 values('1','a',12);"""
+    sql """insert into test_prune3 values('1','a',12);"""
+
+    qt_select """select t3.id from test_prune1 t1 inner join test_prune2 t2 on true inner join test_prune3 t3 on t3.id = t2.id;"""
+
+    explain {
+        sql("select count(*) from test_prune1 where id = '1';")
+        notContains "age"
+    }
+
+    sql """ DROP TABLE IF EXISTS `test_prune1` """
+    sql """ DROP TABLE IF EXISTS `test_prune2` """
+    sql """ DROP TABLE IF EXISTS `test_prune3` """
+}


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