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:05:02 UTC

[doris] 01/08: [Fix](Nereids) Check bound status in analyze straight after bounding (#18581)

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

morningman pushed a commit to branch branch-2.0-alpha
in repository https://gitbox.apache.org/repos/asf/doris.git

commit cf3f04567d3ec74e00532f50f75d3d39989a0f97
Author: LiBinfeng <46...@users.noreply.github.com>
AuthorDate: Thu Apr 20 18:50:13 2023 +0800

    [Fix](Nereids) Check bound status in analyze straight  after bounding (#18581)
    
    Probleam:
    Dead loop cause of keep pushing analyze tasks into job stack. When doing analyze process and generate new operators, the same analyze rule would be pushed again, so it cause dead loop. And analyze process generate new operators when trying to bound order by key and aggregate function.
    
    Solve:
    We need to make it throw exception before complex analyze and rewrite process, so checking whether all expressions being bound should be done twice. One is done after bounding all expression, another is done after all analyze process in case of generate new expressions and new operators.
    
    Example:
    Cases were put in file: regression-test/suites/nereids_p0/except/test_bound_exception.groovy
---
 .../doris/nereids/analyzer/NereidsAnalyzer.java    |  4 ++
 .../org/apache/doris/nereids/rules/RuleType.java   |  1 +
 .../nereids/rules/analysis/CheckAnalysis.java      | 24 +++----
 .../doris/nereids/rules/analysis/CheckBound.java   | 74 ++++++++++++++++++++++
 .../rules/analysis/BindSlotReferenceTest.java      |  2 +-
 .../nereids_p0/except/test_bound_exception.groovy  | 55 ++++++++++++++++
 6 files changed, 148 insertions(+), 12 deletions(-)

diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/analyzer/NereidsAnalyzer.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/analyzer/NereidsAnalyzer.java
index 38d75e2a01..2413e775c1 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/analyzer/NereidsAnalyzer.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/analyzer/NereidsAnalyzer.java
@@ -24,6 +24,7 @@ import org.apache.doris.nereids.rules.analysis.AdjustAggregateNullableForEmptySe
 import org.apache.doris.nereids.rules.analysis.BindExpression;
 import org.apache.doris.nereids.rules.analysis.BindRelation;
 import org.apache.doris.nereids.rules.analysis.CheckAnalysis;
+import org.apache.doris.nereids.rules.analysis.CheckBound;
 import org.apache.doris.nereids.rules.analysis.CheckPolicy;
 import org.apache.doris.nereids.rules.analysis.FillUpMissingSlots;
 import org.apache.doris.nereids.rules.analysis.NormalizeRepeat;
@@ -53,6 +54,9 @@ public class NereidsAnalyzer extends BatchRewriteJob {
                 new UserAuthentication(),
                 new BindExpression()
             ),
+            bottomUp(
+                new CheckBound()
+            ),
             bottomUp(
                 new ProjectToGlobalAggregate(),
                 // this rule check's the logicalProject node's isDisinct property
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 9406267fc4..1397c10153 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
@@ -85,6 +85,7 @@ public enum RuleType {
     // check analysis rule
     CHECK_AGGREGATE_ANALYSIS(RuleTypeClass.CHECK),
     CHECK_ANALYSIS(RuleTypeClass.CHECK),
+    CHECK_BOUND(RuleTypeClass.CHECK),
     CHECK_DATATYPES(RuleTypeClass.CHECK),
 
     // rewrite rules
diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAnalysis.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAnalysis.java
index 8d45087b84..c16f7ad460 100644
--- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAnalysis.java
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckAnalysis.java
@@ -78,17 +78,19 @@ public class CheckAnalysis implements AnalysisRuleFactory {
                 .flatMap(Set::stream)
                 .collect(Collectors.toSet());
         if (!unbounds.isEmpty()) {
-            throw new AnalysisException(String.format("unbounded object %s.",
-                    StringUtils.join(unbounds.stream()
-                            .map(unbound -> {
-                                if (unbound instanceof UnboundSlot) {
-                                    return ((UnboundSlot) unbound).toSql();
-                                } else if (unbound instanceof UnboundFunction) {
-                                    return ((UnboundFunction) unbound).toSql();
-                                }
-                                return unbound.toString();
-                            })
-                            .collect(Collectors.toSet()), ", ")));
+            throw new AnalysisException(String.format("unbounded object %s in %s clause.",
+                StringUtils.join(unbounds.stream()
+                    .map(unbound -> {
+                        if (unbound instanceof UnboundSlot) {
+                            return ((UnboundSlot) unbound).toSql();
+                        } else if (unbound instanceof UnboundFunction) {
+                            return ((UnboundFunction) unbound).toSql();
+                        }
+                        return unbound.toString();
+                    })
+                    .collect(Collectors.toSet()), ", "),
+                plan.getType().toString().substring("LOGICAL_".length())
+            ));
         }
     }
 
diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckBound.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckBound.java
new file mode 100644
index 0000000000..e04ff2c680
--- /dev/null
+++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckBound.java
@@ -0,0 +1,74 @@
+// 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.analysis;
+
+import org.apache.doris.nereids.analyzer.Unbound;
+import org.apache.doris.nereids.analyzer.UnboundFunction;
+import org.apache.doris.nereids.analyzer.UnboundSlot;
+import org.apache.doris.nereids.exceptions.AnalysisException;
+import org.apache.doris.nereids.rules.Rule;
+import org.apache.doris.nereids.rules.RuleType;
+import org.apache.doris.nereids.trees.plans.Plan;
+
+import com.google.common.collect.ImmutableList;
+import org.apache.commons.lang3.StringUtils;
+
+import java.util.List;
+import java.util.Set;
+import java.util.stream.Collectors;
+
+/**
+ * Check bound rule to check semantic correct after bounding of expression by Nereids.
+ * Also give operator information without LOGICAL_
+ */
+public class CheckBound implements AnalysisRuleFactory {
+
+    @Override
+    public List<Rule> buildRules() {
+        return ImmutableList.of(
+            RuleType.CHECK_BOUND.build(
+                any().then(plan -> {
+                    checkBound(plan);
+                    return null;
+                })
+            )
+        );
+    }
+
+    private void checkBound(Plan plan) {
+        Set<Unbound> unbounds = plan.getExpressions().stream()
+                .<Set<Unbound>>map(e -> e.collect(Unbound.class::isInstance))
+                .flatMap(Set::stream)
+                .collect(Collectors.toSet());
+        if (!unbounds.isEmpty()) {
+            throw new AnalysisException(String.format("unbounded object %s in %s clause.",
+                StringUtils.join(unbounds.stream()
+                    .map(unbound -> {
+                        if (unbound instanceof UnboundSlot) {
+                            return ((UnboundSlot) unbound).toSql();
+                        } else if (unbound instanceof UnboundFunction) {
+                            return ((UnboundFunction) unbound).toSql();
+                        }
+                        return unbound.toString();
+                    })
+                    .collect(Collectors.toSet()), ", "),
+                    plan.getType().toString().substring("LOGICAL_".length())
+            ));
+        }
+    }
+}
diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/analysis/BindSlotReferenceTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/analysis/BindSlotReferenceTest.java
index ad8ca7bbc5..5ddb7f5643 100644
--- a/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/analysis/BindSlotReferenceTest.java
+++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/rules/analysis/BindSlotReferenceTest.java
@@ -54,7 +54,7 @@ class BindSlotReferenceTest {
                 new LogicalOlapScan(RelationUtil.newRelationId(), PlanConstructor.student));
         AnalysisException exception = Assertions.assertThrows(AnalysisException.class,
                 () -> PlanChecker.from(MemoTestUtils.createConnectContext()).analyze(project));
-        Assertions.assertEquals("unbounded object foo.", exception.getMessage());
+        Assertions.assertEquals("unbounded object foo in PROJECT clause.", exception.getMessage());
     }
 
     @Test
diff --git a/regression-test/suites/nereids_p0/except/test_bound_exception.groovy b/regression-test/suites/nereids_p0/except/test_bound_exception.groovy
new file mode 100644
index 0000000000..95479f009d
--- /dev/null
+++ b/regression-test/suites/nereids_p0/except/test_bound_exception.groovy
@@ -0,0 +1,55 @@
+// 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("test_bound_exception") {
+    sql "SET enable_nereids_planner=true"
+    sql "SET enable_fallback_to_original_planner=false"
+    def tbName = "test_bound_exception"
+    def dbName = "test_bound_db"
+    sql "CREATE DATABASE IF NOT EXISTS ${dbName}"
+    sql "USE ${dbName}"
+
+    sql """ DROP TABLE IF EXISTS ${tbName} """
+    sql """
+        create table if not exists ${tbName} (id int, name char(10))
+        distributed by hash(id) buckets 1 properties("replication_num"="1");
+    """
+    test {
+        sql "SELECT id FROM ${tbName} GROUP BY id ORDER BY id123"
+        exception "errCode = 2, detailMessage = Unexpected exception: unbounded object id123 in SORT clause."
+    }
+    test {
+        sql "SELECT id123 FROM ${tbName} ORDER BY id"
+        exception "errCode = 2, detailMessage = Unexpected exception: unbounded object id123 in PROJECT clause."
+    }
+    test {
+        sql "SELECT id123 FROM ${tbName} GROUP BY id ORDER BY id"
+        exception "errCode = 2, detailMessage = Unexpected exception: unbounded object id123 in AGGREGATE clause."
+    }
+    test {
+        sql "SELECT id FROM ${tbName} GROUP BY id123 ORDER BY id"
+        exception "errCode = 2, detailMessage = Unexpected exception: cannot bind GROUP BY KEY: id123"
+    }
+    test {
+        sql "SELECT id FROM ${tbName} WHERE id = (SELECT id from ${tbName} ORDER BY id123 LIMIT 1) ORDER BY id"
+        exception "errCode = 2, detailMessage = Unexpected exception: unbounded object id123 in SORT clause."
+    }
+    test {
+        sql "SELECT id FROM ${tbName} WHERE id123 = 123 ORDER BY id"
+        exception "errCode = 2, detailMessage = Unexpected exception: Invalid call to dataType on unbound object"
+    }
+}


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