You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by ka...@apache.org on 2020/02/24 02:41:26 UTC

[incubator-doris] branch master updated: Add more constraints for bitmap column (#2966)

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

kangkaisen pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-doris.git


The following commit(s) were added to refs/heads/master by this push:
     new fb5b58b  Add more constraints for bitmap column (#2966)
fb5b58b is described below

commit fb5b58b75afb0077790b316169b7fad3cb97b88b
Author: kangkaisen <ka...@apache.org>
AuthorDate: Mon Feb 24 10:41:18 2020 +0800

    Add more constraints for bitmap column (#2966)
---
 .../apache/doris/analysis/FunctionCallExpr.java    |  36 ++---
 .../org/apache/doris/analysis/GroupByClause.java   |   7 +-
 .../java/org/apache/doris/analysis/SelectStmt.java |   6 +-
 .../main/java/org/apache/doris/catalog/Type.java   |  13 ++
 .../{utframe => planner}/ConstantExpressTest.java  |   3 +-
 .../QueryPlanTest.java}                            | 149 +++++++++++++++++++--
 6 files changed, 168 insertions(+), 46 deletions(-)

diff --git a/fe/src/main/java/org/apache/doris/analysis/FunctionCallExpr.java b/fe/src/main/java/org/apache/doris/analysis/FunctionCallExpr.java
index 3c05234..60c15ec 100644
--- a/fe/src/main/java/org/apache/doris/analysis/FunctionCallExpr.java
+++ b/fe/src/main/java/org/apache/doris/analysis/FunctionCallExpr.java
@@ -225,8 +225,6 @@ public class FunctionCallExpr extends Expr {
         Preconditions.checkNotNull(fn);
         return fn instanceof AggregateFunction
             && ((AggregateFunction) fn).returnsNonNullOnEmpty();
-              //  && !isAnalyticFnCall
-              //  && ((BuiltinAggregateFunction) fn).op().returnsNonNullOnEmpty();
     }
 
     public boolean isDistinct() {
@@ -278,10 +276,9 @@ public class FunctionCallExpr extends Expr {
                         "COUNT must have DISTINCT for multiple arguments: " + this.toSql());
             }
 
-            for (int i = 0; i < children.size(); i++) {
-                if (children.get(i).type.isHllType()) {
-                    throw new AnalysisException(
-                            "hll only use in HLL_UNION_AGG or HLL_CARDINALITY , HLL_HASH and so on.");
+            for (Expr child : children) {
+                if (child.type.isOnlyMetricType()) {
+                    throw new AnalysisException(Type.OnlyMetricTypeErrorMsg);
                 }
             }
             return;
@@ -303,22 +300,12 @@ public class FunctionCallExpr extends Expr {
                          "group_concat requires first parameter to be of type STRING: " + this.toSql());
             }
 
-            if (arg0.type.isHllType()) {
-                throw new AnalysisException(
-                         "group_concat requires first parameter can't be of type HLL: " + this.toSql());
-            }
-
             if (children.size() == 2) {
                 Expr arg1 = getChild(1);
                 if (!arg1.type.isStringType() && !arg1.type.isNull()) {
                     throw new AnalysisException(
                             "group_concat requires second parameter to be of type STRING: " + this.toSql());
                 }
-
-                if (arg0.type.isHllType()) {
-                    throw new AnalysisException(
-                            "group_concat requires second parameter can't be of type HLL: " + this.toSql());
-                }
             }
             return;
         }
@@ -326,7 +313,7 @@ public class FunctionCallExpr extends Expr {
         if (fnName.getFunction().equalsIgnoreCase("lag")
                 || fnName.getFunction().equalsIgnoreCase("lead")) {
             if (!isAnalyticFnCall) {
-                new AnalysisException(fnName.getFunction() + " only used in analytic function");
+                throw new AnalysisException(fnName.getFunction() + " only used in analytic function");
             } else {
                 if (children.size() > 2) {
                     if (!getChild(2).isConstant()) {
@@ -346,7 +333,7 @@ public class FunctionCallExpr extends Expr {
                 || fnName.getFunction().equalsIgnoreCase("last_value")
                 || fnName.getFunction().equalsIgnoreCase("first_value_rewrite")) {
             if (!isAnalyticFnCall) {
-                new AnalysisException(fnName.getFunction() + " only used in analytic function");
+                throw new AnalysisException(fnName.getFunction() + " only used in analytic function");
             }
         }
 
@@ -359,11 +346,11 @@ public class FunctionCallExpr extends Expr {
         // SUM and AVG cannot be applied to non-numeric types
         if ((fnName.getFunction().equalsIgnoreCase("sum")
                 || fnName.getFunction().equalsIgnoreCase("avg"))
-                && ((!arg.type.isNumericType() && !arg.type.isNull()) || arg.type.isHllType())) {
+                && ((!arg.type.isNumericType() && !arg.type.isNull()) || arg.type.isOnlyMetricType())) {
             throw new AnalysisException(fnName.getFunction() + " requires a numeric parameter: " + this.toSql());
         }
         if (fnName.getFunction().equalsIgnoreCase("sum_distinct")
-                && ((!arg.type.isNumericType() && !arg.type.isNull()) || arg.type.isHllType())) {
+                && ((!arg.type.isNumericType() && !arg.type.isNull()) || arg.type.isOnlyMetricType())) {
             throw new AnalysisException(
                     "SUM_DISTINCT requires a numeric parameter: " + this.toSql());
         }
@@ -373,9 +360,8 @@ public class FunctionCallExpr extends Expr {
                 || fnName.getFunction().equalsIgnoreCase("DISTINCT_PC")
                 || fnName.getFunction().equalsIgnoreCase("DISTINCT_PCSA")
                 || fnName.getFunction().equalsIgnoreCase("NDV"))
-                && arg.type.isHllType()) {
-            throw new AnalysisException(
-                    "hll only use in HLL_UNION_AGG or HLL_CARDINALITY , HLL_HASH and so on.");
+                && arg.type.isOnlyMetricType()) {
+            throw new AnalysisException(Type.OnlyMetricTypeErrorMsg);
         }
 
         if ((fnName.getFunction().equalsIgnoreCase(FunctionSet.BITMAP_UNION_INT) && !arg.type.isInteger32Type())) {
@@ -446,7 +432,6 @@ public class FunctionCallExpr extends Expr {
                             + this.toSql());
                 }
             }
-            return;
         }
     }
 
@@ -656,7 +641,6 @@ public class FunctionCallExpr extends Expr {
         // Inherit the function object from 'agg'.
         result.fn = agg.fn;
         result.type = agg.type;
-        // Preconditions.checkState(!result.type.isWildcardDecimal());
         return result;
     }
 
@@ -711,7 +695,7 @@ public class FunctionCallExpr extends Expr {
         return super.isConstantImpl();
     }
 
-    static boolean isNondeterministicBuiltinFnName(String fnName) {
+    private static boolean isNondeterministicBuiltinFnName(String fnName) {
         if (fnName.equalsIgnoreCase("rand") || fnName.equalsIgnoreCase("random")
                 || fnName.equalsIgnoreCase("uuid")) {
             return true;
diff --git a/fe/src/main/java/org/apache/doris/analysis/GroupByClause.java b/fe/src/main/java/org/apache/doris/analysis/GroupByClause.java
index fcde852..005d882 100644
--- a/fe/src/main/java/org/apache/doris/analysis/GroupByClause.java
+++ b/fe/src/main/java/org/apache/doris/analysis/GroupByClause.java
@@ -17,6 +17,7 @@
 
 package org.apache.doris.analysis;
 
+import org.apache.doris.catalog.Type;
 import org.apache.doris.common.AnalysisException;
 
 import com.google.common.base.Preconditions;
@@ -185,10 +186,8 @@ public class GroupByClause implements ParseNode {
                                 + groupingExpr.toSql());
             }
 
-            if (groupingExpr.type.isHllType()) {
-                throw new AnalysisException(
-                        "GROUP BY expression must not contain hll column: "
-                                + groupingExpr.toSql());
+            if (groupingExpr.type.isOnlyMetricType()) {
+                throw new AnalysisException(Type.OnlyMetricTypeErrorMsg);
             }
         }
 
diff --git a/fe/src/main/java/org/apache/doris/analysis/SelectStmt.java b/fe/src/main/java/org/apache/doris/analysis/SelectStmt.java
index 05de06c..e4f5299 100644
--- a/fe/src/main/java/org/apache/doris/analysis/SelectStmt.java
+++ b/fe/src/main/java/org/apache/doris/analysis/SelectStmt.java
@@ -789,12 +789,10 @@ public class SelectStmt extends QueryStmt {
     private void expandStar(TableName tblName, TupleDescriptor desc) throws AnalysisException {
         for (Column col : desc.getTable().getBaseSchema()) {
             if (col.getDataType() == PrimitiveType.HLL && !fromInsert) {
-                throw new AnalysisException(
-                        "hll only use in HLL_UNION_AGG or HLL_CARDINALITY , HLL_HASH and so on.");
+                throw new AnalysisException(Type.OnlyMetricTypeErrorMsg);
             }
             if (col.getAggregationType() == AggregateType.BITMAP_UNION && !fromInsert) {
-                throw new AnalysisException(
-                        "BITMAP_UNION agg column only use in TO_BITMAP or BITMAP_UNION , BITMAP_COUNT and so on.");
+                throw new AnalysisException(Type.OnlyMetricTypeErrorMsg);
             }
             resultExprs.add(new SlotRef(tblName, col.getName()));
             colLabels.add(col.getName());
diff --git a/fe/src/main/java/org/apache/doris/catalog/Type.java b/fe/src/main/java/org/apache/doris/catalog/Type.java
index 2f4fc53..e2d8119 100644
--- a/fe/src/main/java/org/apache/doris/catalog/Type.java
+++ b/fe/src/main/java/org/apache/doris/catalog/Type.java
@@ -189,6 +189,19 @@ public abstract class Type {
         return isScalarType(PrimitiveType.VARCHAR) || isScalarType(PrimitiveType.CHAR);
     }
 
+    // only metric types have the following constraint:
+    // 1. don't support as key column
+    // 2. don't support filter
+    // 3. don't support group by
+    // 4. don't support index
+    public boolean isOnlyMetricType() {
+        return isScalarType(PrimitiveType.HLL) || isScalarType(PrimitiveType.BITMAP);
+    }
+
+    public static final String OnlyMetricTypeErrorMsg =
+            "Doris hll and bitmap column must use with specific function, and don't support filter or group by." +
+                    "please run 'help hll' or 'help bitmap' in your mysql client.";
+
     public boolean isHllType() {
         return isScalarType(PrimitiveType.HLL);
     }
diff --git a/fe/src/test/java/org/apache/doris/utframe/ConstantExpressTest.java b/fe/src/test/java/org/apache/doris/planner/ConstantExpressTest.java
similarity index 98%
rename from fe/src/test/java/org/apache/doris/utframe/ConstantExpressTest.java
rename to fe/src/test/java/org/apache/doris/planner/ConstantExpressTest.java
index f3ff51d..38b20c7 100644
--- a/fe/src/test/java/org/apache/doris/utframe/ConstantExpressTest.java
+++ b/fe/src/test/java/org/apache/doris/planner/ConstantExpressTest.java
@@ -15,9 +15,10 @@
 // specific language governing permissions and limitations
 // under the License.
 
-package org.apache.doris.utframe;
+package org.apache.doris.planner;
 
 import org.apache.doris.qe.ConnectContext;
+import org.apache.doris.utframe.UtFrameUtils;
 import org.junit.Assert;
 import org.junit.BeforeClass;
 import org.junit.Test;
diff --git a/fe/src/test/java/org/apache/doris/utframe/BitmapFunctionTest.java b/fe/src/test/java/org/apache/doris/planner/QueryPlanTest.java
similarity index 52%
rename from fe/src/test/java/org/apache/doris/utframe/BitmapFunctionTest.java
rename to fe/src/test/java/org/apache/doris/planner/QueryPlanTest.java
index b8c12cb..0c6afa8 100644
--- a/fe/src/test/java/org/apache/doris/utframe/BitmapFunctionTest.java
+++ b/fe/src/test/java/org/apache/doris/planner/QueryPlanTest.java
@@ -15,25 +15,29 @@
 // specific language governing permissions and limitations
 // under the License.
 
-package org.apache.doris.utframe;
+package org.apache.doris.planner;
+
 
 import org.apache.doris.analysis.CreateDbStmt;
 import org.apache.doris.analysis.CreateTableStmt;
 import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.Type;
 import org.apache.doris.qe.ConnectContext;
 
+import org.apache.doris.utframe.UtFrameUtils;
 import org.junit.Assert;
 import org.junit.BeforeClass;
 import org.junit.Test;
 
 import java.util.UUID;
 
-public class BitmapFunctionTest {
+public class QueryPlanTest {
     // use a unique dir so that it won't be conflict with other unit test which
     // may also start a Mocked Frontend
-    private static String runningDir = "fe/mocked/BitmapFunctionTest/" + UUID.randomUUID().toString() + "/";
+    private static String runningDir = "fe/mocked/QueryPlanTest/" + UUID.randomUUID().toString() + "/";
 
     private static ConnectContext connectContext;
+
     @BeforeClass
     public static void beforeClass() throws Exception {
         UtFrameUtils.createMinDorisCluster(runningDir);
@@ -44,8 +48,8 @@ public class BitmapFunctionTest {
         String createDbStmtStr = "create database test;";
         CreateDbStmt createDbStmt = (CreateDbStmt) UtFrameUtils.parseAndAnalyzeStmt(createDbStmtStr, connectContext);
         Catalog.getCurrentCatalog().createDb(createDbStmt);
-        // create table
-        String createTblStmtStr = "CREATE TABLE test.bitmap_table (\n" +
+
+        createTable("CREATE TABLE test.bitmap_table (\n" +
                 "  `id` int(11) NULL COMMENT \"\",\n" +
                 "  `id2` bitmap bitmap_union NULL\n" +
                 ") ENGINE=OLAP\n" +
@@ -53,11 +57,9 @@ public class BitmapFunctionTest {
                 "DISTRIBUTED BY HASH(`id`) BUCKETS 1\n" +
                 "PROPERTIES (\n" +
                 " \"replication_num\" = \"1\"\n" +
-                ");";
-        CreateTableStmt createTableStmt = (CreateTableStmt) UtFrameUtils.parseAndAnalyzeStmt(createTblStmtStr, connectContext);
-        Catalog.getCurrentCatalog().createTable(createTableStmt);
+                ");");
 
-        createTblStmtStr = "CREATE TABLE test.bitmap_table_2 (\n" +
+        createTable("CREATE TABLE test.bitmap_table_2 (\n" +
                 "  `id` int(11) NULL COMMENT \"\",\n" +
                 "  `id2` bitmap bitmap_union NULL\n" +
                 ") ENGINE=OLAP\n" +
@@ -65,8 +67,21 @@ public class BitmapFunctionTest {
                 "DISTRIBUTED BY HASH(`id`) BUCKETS 1\n" +
                 "PROPERTIES (\n" +
                 " \"replication_num\" = \"1\"\n" +
-                ");";
-        createTableStmt = (CreateTableStmt) UtFrameUtils.parseAndAnalyzeStmt(createTblStmtStr, connectContext);
+                ");");
+
+        createTable("CREATE TABLE test.hll_table (\n" +
+                "  `id` int(11) NULL COMMENT \"\",\n" +
+                "  `id2` hll hll_union NULL\n" +
+                ") ENGINE=OLAP\n" +
+                "AGGREGATE KEY(`id`)\n" +
+                "DISTRIBUTED BY HASH(`id`) BUCKETS 1\n" +
+                "PROPERTIES (\n" +
+                " \"replication_num\" = \"1\"\n" +
+                ");");
+    }
+
+    private static void createTable(String sql) throws Exception {
+        CreateTableStmt createTableStmt = (CreateTableStmt) UtFrameUtils.parseAndAnalyzeStmt(sql, connectContext);
         Catalog.getCurrentCatalog().createTable(createTableStmt);
     }
 
@@ -105,4 +120,116 @@ public class BitmapFunctionTest {
         String errorMsg = UtFrameUtils.getSQLPlanOrErrorMsg(connectContext, queryStr);
         Assert.assertTrue(errorMsg.contains("bitmap column id2 require the function return type is BITMAP"));
     }
+
+    private static void testBitmapQueryPlan(String sql, String result) throws Exception {
+        String explainString = UtFrameUtils.getSQLPlanOrErrorMsg(connectContext, "explain " + sql);
+        Assert.assertTrue(explainString.contains(result));
+    }
+
+    @Test
+    public void testBitmapQuery() throws Exception {
+        testBitmapQueryPlan(
+                "select * from test.bitmap_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testBitmapQueryPlan(
+                "select count(id2) from test.bitmap_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testBitmapQueryPlan(
+                "select group_concat(id2) from test.bitmap_table;",
+                "group_concat requires first parameter to be of type STRING: group_concat(`id2`)"
+        );
+
+        testBitmapQueryPlan(
+                "select sum(id2) from test.bitmap_table;",
+                "sum requires a numeric parameter: sum(`id2`)"
+        );
+
+        testBitmapQueryPlan(
+                "select avg(id2) from test.bitmap_table;",
+                "avg requires a numeric parameter: avg(`id2`)"
+        );
+
+        testBitmapQueryPlan(
+                "select max(id2) from test.bitmap_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testBitmapQueryPlan(
+                "select min(id2) from test.bitmap_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testBitmapQueryPlan(
+                "select count(*) from test.bitmap_table group by id2;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testBitmapQueryPlan(
+                "select count(*) from test.bitmap_table where id2 = 1;",
+                "type not match, originType=BITMAP, targeType=DOUBLE"
+        );
+
+    }
+
+    private static void testHLLQueryPlan(String sql, String result) throws Exception {
+        String explainString = UtFrameUtils.getSQLPlanOrErrorMsg(connectContext, "explain " + sql);
+        Assert.assertTrue(explainString.contains(result));
+    }
+
+    @Test
+    public void testHLLTypeQuery() throws Exception {
+        testHLLQueryPlan(
+                "select * from test.hll_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testHLLQueryPlan(
+                "select count(id2) from test.hll_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testHLLQueryPlan(
+                "select group_concat(id2) from test.hll_table;",
+                "group_concat requires first parameter to be of type STRING: group_concat(`id2`)"
+        );
+
+        testHLLQueryPlan(
+                "select sum(id2) from test.hll_table;",
+                "sum requires a numeric parameter: sum(`id2`)"
+        );
+
+        testHLLQueryPlan(
+                "select avg(id2) from test.hll_table;",
+                "avg requires a numeric parameter: avg(`id2`)"
+        );
+
+        testHLLQueryPlan(
+                "select max(id2) from test.hll_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testHLLQueryPlan(
+                "select min(id2) from test.hll_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testHLLQueryPlan(
+                "select min(id2) from test.hll_table;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testHLLQueryPlan(
+                "select count(*) from test.hll_table group by id2;",
+                Type.OnlyMetricTypeErrorMsg
+        );
+
+        testHLLQueryPlan(
+                "select count(*) from test.hll_table where id2 = 1",
+                "type not match, originType=HLL, targeType=DOUBLE"
+        );
+    }
 }


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