This is an automated email from the ASF dual-hosted git repository. yiguolei pushed a commit to branch branch-2.1 in repository https://gitbox.apache.org/repos/asf/doris.git
commit ffa8167b369de5df71e0089dd7ba1e72eab959bd Author: zhangdong <493738...@qq.com> AuthorDate: Wed Feb 7 11:40:18 2024 +0800 [enhance](mtmv) Limit the number of partitions for table creation (#30867) - Creating too many partitions is time-consuming, so limiting the number of partitions - add more case,such as `mor`,`mow` --- .../org/apache/doris/analysis/PartitionDesc.java | 9 +++ .../org/apache/doris/mtmv/MTMVPartitionUtil.java | 27 ++++++++ .../java/org/apache/doris/mtmv/MTMVService.java | 4 -- .../trees/plans/commands/info/CreateMTMVInfo.java | 28 ++++++-- .../trees/plans/commands/info/CreateTableInfo.java | 11 ++++ .../java/org/apache/doris/qe/SessionVariable.java | 12 ++++ .../data/mtmv_p0/test_agg_table_mtmv.out | 4 ++ .../data/mtmv_p0/test_mor_table_mtmv.out | 4 ++ .../data/mtmv_p0/test_mow_table_mtmv.out | 4 ++ .../suites/mtmv_p0/test_agg_table_mtmv.groovy | 62 ++++++++++++++++++ .../suites/mtmv_p0/test_mor_table_mtmv.groovy | 62 ++++++++++++++++++ .../suites/mtmv_p0/test_mow_table_mtmv.groovy | 62 ++++++++++++++++++ .../mtmv_p0/test_partition_limit_mtmv.groovy | 74 ++++++++++++++++++++++ .../mtmv_p0/test_partition_refresh_mtmv.groovy | 2 + 14 files changed, 355 insertions(+), 10 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/analysis/PartitionDesc.java b/fe/fe-core/src/main/java/org/apache/doris/analysis/PartitionDesc.java index 625e0de8228..1f68bc2c6b3 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/analysis/PartitionDesc.java +++ b/fe/fe-core/src/main/java/org/apache/doris/analysis/PartitionDesc.java @@ -159,6 +159,15 @@ public class PartitionDesc { throw new AnalysisException("No partition columns."); } + int createTablePartitionMaxNum = ConnectContext.get().getSessionVariable().getCreateTablePartitionMaxNum(); + if (singlePartitionDescs.size() > createTablePartitionMaxNum) { + throw new AnalysisException(String.format( + "The number of partitions to be created is [%s], exceeding the maximum value of [%s]. " + + "Creating too many partitions can be time-consuming. If necessary, " + + "You can set the session variable 'create_table_partition_max_num' to a larger value.", + singlePartitionDescs.size(), createTablePartitionMaxNum)); + } + // `analyzeUniqueKeyMergeOnWrite` would modify `properties`, which will be used later, // so we just clone a properties map here. boolean enableUniqueKeyMergeOnWrite = false; diff --git a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPartitionUtil.java b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPartitionUtil.java index a8e3e11869d..bcf182e4b1f 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPartitionUtil.java +++ b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVPartitionUtil.java @@ -18,6 +18,7 @@ package org.apache.doris.mtmv; import org.apache.doris.analysis.AddPartitionClause; +import org.apache.doris.analysis.AllPartitionDesc; import org.apache.doris.analysis.DropPartitionClause; import org.apache.doris.analysis.PartitionKeyDesc; import org.apache.doris.analysis.SinglePartitionDesc; @@ -38,6 +39,7 @@ import org.apache.logging.log4j.LogManager; import org.apache.logging.log4j.Logger; import java.util.Collection; +import java.util.HashMap; import java.util.List; import java.util.Map; import java.util.Map.Entry; @@ -106,6 +108,31 @@ public class MTMVPartitionUtil { } } + /** + * getPartitionDescsByRelatedTable when create MTMV + * + * @param relatedTable + * @param tableProperties + * @return + * @throws AnalysisException + */ + public static List<AllPartitionDesc> getPartitionDescsByRelatedTable(MTMVRelatedTableIf relatedTable, + Map<String, String> tableProperties) throws AnalysisException { + HashMap<String, String> partitionProperties = Maps.newHashMap(); + List<AllPartitionDesc> res = Lists.newArrayList(); + Map<Long, PartitionItem> relatedTableItems = relatedTable.getPartitionItems(); + for (Entry<Long, PartitionItem> entry : relatedTableItems.entrySet()) { + PartitionKeyDesc oldPartitionKeyDesc = entry.getValue().toPartitionKeyDesc(); + SinglePartitionDesc singlePartitionDesc = new SinglePartitionDesc(true, + generatePartitionName(oldPartitionKeyDesc), + oldPartitionKeyDesc, partitionProperties); + // mtmv can only has one partition col + singlePartitionDesc.analyze(1, tableProperties); + res.add(singlePartitionDesc); + } + return res; + } + public static List<String> getPartitionNamesByIds(MTMV mtmv, Collection<Long> ids) throws AnalysisException { List<String> res = Lists.newArrayList(); for (Long partitionId : ids) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVService.java b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVService.java index 227166e56d3..cbbaef6b917 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVService.java +++ b/fe/fe-core/src/main/java/org/apache/doris/mtmv/MTMVService.java @@ -24,7 +24,6 @@ import org.apache.doris.common.DdlException; import org.apache.doris.common.MetaNotFoundException; import org.apache.doris.job.exception.JobException; import org.apache.doris.job.extensions.mtmv.MTMVTask; -import org.apache.doris.mtmv.MTMVPartitionInfo.MTMVPartitionType; import org.apache.doris.nereids.trees.plans.commands.info.CancelMTMVTaskInfo; import org.apache.doris.nereids.trees.plans.commands.info.PauseMTMVInfo; import org.apache.doris.nereids.trees.plans.commands.info.RefreshMTMVInfo; @@ -85,9 +84,6 @@ public class MTMVService { public void createMTMV(MTMV mtmv) throws DdlException, AnalysisException { Objects.requireNonNull(mtmv); LOG.info("createMTMV: " + mtmv.getName()); - if (mtmv.getMvPartitionInfo().getPartitionType() == MTMVPartitionType.FOLLOW_BASE_TABLE) { - MTMVPartitionUtil.alignMvPartition(mtmv, mtmv.getMvPartitionInfo().getRelatedTable()); - } for (MTMVHookService mtmvHookService : hooks.values()) { mtmvHookService.createMTMV(mtmv); } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java index db9abb8f02c..2b2fe8ab9f2 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateMTMVInfo.java @@ -17,6 +17,7 @@ package org.apache.doris.nereids.trees.plans.commands.info; +import org.apache.doris.analysis.AllPartitionDesc; import org.apache.doris.analysis.CreateMTMVStmt; import org.apache.doris.analysis.KeysDesc; import org.apache.doris.analysis.ListPartitionDesc; @@ -37,6 +38,7 @@ import org.apache.doris.common.util.PropertyAnalyzer; import org.apache.doris.mtmv.EnvInfo; import org.apache.doris.mtmv.MTMVPartitionInfo; import org.apache.doris.mtmv.MTMVPartitionInfo.MTMVPartitionType; +import org.apache.doris.mtmv.MTMVPartitionUtil; import org.apache.doris.mtmv.MTMVPlanUtil; import org.apache.doris.mtmv.MTMVRefreshInfo; import org.apache.doris.mtmv.MTMVRelatedTableIf; @@ -241,7 +243,7 @@ public class CreateMTMVInfo { } getRelation(planner); getColumns(plan); - analyzePartition(planner); + analyzePartition(planner, ctx); } private void getRelation(NereidsPlanner planner) { @@ -264,7 +266,7 @@ public class CreateMTMVInfo { this.relation = MTMVPlanUtil.generateMTMVRelation(plan); } - private void analyzePartition(NereidsPlanner planner) { + private void analyzePartition(NereidsPlanner planner, ConnectContext ctx) { if (mvPartitionInfo.getPartitionType() == MTMVPartitionType.FOLLOW_BASE_TABLE) { CascadesContext cascadesContext = planner.getCascadesContext(); @@ -307,7 +309,7 @@ public class CreateMTMVInfo { } mvPartitionInfo.setRelatedTable(relatedTableInfo.get().getTableInfo()); mvPartitionInfo.setRelatedCol(relatedTableInfo.get().getColumn()); - partitionDesc = generatePartitionDesc(mtmvBaseRealtedTable); + partitionDesc = generatePartitionDesc(mtmvBaseRealtedTable, ctx); } finally { // after operate, roll back the disable rules sessionVariable.setDisableNereidsRules(String.join(",", tempDisableRules)); @@ -316,15 +318,29 @@ public class CreateMTMVInfo { } } - private PartitionDesc generatePartitionDesc(MTMVRelatedTableIf relatedTable) { + private PartitionDesc generatePartitionDesc(MTMVRelatedTableIf relatedTable, ConnectContext ctx) { + List<AllPartitionDesc> allPartitionDescs = null; + try { + allPartitionDescs = MTMVPartitionUtil + .getPartitionDescsByRelatedTable(relatedTable, properties); + } catch (org.apache.doris.common.AnalysisException e) { + throw new AnalysisException("getPartitionDescsByRelatedTable failed", e); + } + if (allPartitionDescs.size() > ctx.getSessionVariable().getCreateTablePartitionMaxNum()) { + throw new AnalysisException(String.format( + "The number of partitions to be created is [%s], exceeding the maximum value of [%s]. " + + "Creating too many partitions can be time-consuming. If necessary, " + + "You can set the session variable 'create_table_partition_max_num' to a larger value.", + allPartitionDescs.size(), ctx.getSessionVariable().getCreateTablePartitionMaxNum())); + } try { PartitionType type = relatedTable.getPartitionType(); if (type == PartitionType.RANGE) { return new RangePartitionDesc(Lists.newArrayList(mvPartitionInfo.getPartitionCol()), - Lists.newArrayList()); + allPartitionDescs); } else if (type == PartitionType.LIST) { return new ListPartitionDesc(Lists.newArrayList(mvPartitionInfo.getPartitionCol()), - Lists.newArrayList()); + allPartitionDescs); } else { return null; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateTableInfo.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateTableInfo.java index 8353ccd308d..296a5c71ec8 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateTableInfo.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/CreateTableInfo.java @@ -813,6 +813,17 @@ public class CreateTableInfo { ? partitions.stream().map(PartitionDefinition::translateToCatalogStyle) .collect(Collectors.toList()) : null; + + int createTablePartitionMaxNum = ConnectContext.get().getSessionVariable().getCreateTablePartitionMaxNum(); + if (partitionDescs != null && partitionDescs.size() > createTablePartitionMaxNum) { + throw new org.apache.doris.nereids.exceptions.AnalysisException(String.format( + "The number of partitions to be created is [%s], exceeding the maximum value of [%s]. " + + "Creating too many partitions can be time-consuming. If necessary, " + + "You can set the session variable 'create_table_partition_max_num' " + + "to a larger value.", + partitionDescs.size(), createTablePartitionMaxNum)); + } + try { if (partitionType.equals(PartitionType.RANGE.name())) { if (isAutoPartition) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java b/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java index 88591a0206e..72ae245636b 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java +++ b/fe/fe-core/src/main/java/org/apache/doris/qe/SessionVariable.java @@ -488,6 +488,9 @@ public class SessionVariable implements Serializable, Writable { public static final String MATERIALIZED_VIEW_REWRITE_ENABLE_CONTAIN_EXTERNAL_TABLE = "materialized_view_rewrite_enable_contain_external_table"; + public static final String CREATE_TABLE_PARTITION_MAX_NUM + = "create_table_partition_max_num"; + public static final String ENABLE_PUSHDOWN_MINMAX_ON_UNIQUE = "enable_pushdown_minmax_on_unique"; public static final String ENABLE_PUSHDOWN_STRING_MINMAX = "enable_pushdown_string_minmax"; @@ -1539,6 +1542,11 @@ public class SessionVariable implements Serializable, Writable { + "when using rewriting based on struct info"}) public boolean materializedViewRewriteEnableContainExternalTable = false; + @VariableMgr.VarAttr(name = CREATE_TABLE_PARTITION_MAX_NUM, needForward = true, + description = {"建表时创建分区的最大数量", + "The maximum number of partitions created during table creation"}) + public int createTablePartitionMaxNum = 10000; + @VariableMgr.VarAttr(name = FORCE_JNI_SCANNER, description = {"强制使用jni方式读取外表", "Force the use of jni mode to read external table"}) private boolean forceJniScanner = false; @@ -3293,6 +3301,10 @@ public class SessionVariable implements Serializable, Writable { return materializedViewRewriteEnableContainExternalTable; } + public int getCreateTablePartitionMaxNum() { + return createTablePartitionMaxNum; + } + public boolean isIgnoreStorageDataDistribution() { return ignoreStorageDataDistribution && getEnablePipelineXEngine() && enableLocalShuffle && enableNereidsPlanner; diff --git a/regression-test/data/mtmv_p0/test_agg_table_mtmv.out b/regression-test/data/mtmv_p0/test_agg_table_mtmv.out new file mode 100644 index 00000000000..c91285acccd --- /dev/null +++ b/regression-test/data/mtmv_p0/test_agg_table_mtmv.out @@ -0,0 +1,4 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +1 3 + diff --git a/regression-test/data/mtmv_p0/test_mor_table_mtmv.out b/regression-test/data/mtmv_p0/test_mor_table_mtmv.out new file mode 100644 index 00000000000..c6c90e1812d --- /dev/null +++ b/regression-test/data/mtmv_p0/test_mor_table_mtmv.out @@ -0,0 +1,4 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +1 2 + diff --git a/regression-test/data/mtmv_p0/test_mow_table_mtmv.out b/regression-test/data/mtmv_p0/test_mow_table_mtmv.out new file mode 100644 index 00000000000..c6c90e1812d --- /dev/null +++ b/regression-test/data/mtmv_p0/test_mow_table_mtmv.out @@ -0,0 +1,4 @@ +-- This file is automatically generated. You should know what you did if you want to edit this +-- !select -- +1 2 + diff --git a/regression-test/suites/mtmv_p0/test_agg_table_mtmv.groovy b/regression-test/suites/mtmv_p0/test_agg_table_mtmv.groovy new file mode 100644 index 00000000000..192a7f0cfea --- /dev/null +++ b/regression-test/suites/mtmv_p0/test_agg_table_mtmv.groovy @@ -0,0 +1,62 @@ +// 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. + +import org.junit.Assert; + +suite("test_agg_table_mtmv") { + def tableName = "t_test_agg_table_mtmv_user" + def mvName = "test_agg_table_mtmv" + def dbName = "regression_test_mtmv_p0" + sql """drop table if exists `${tableName}`""" + sql """drop materialized view if exists ${mvName};""" + + sql """ + CREATE TABLE `${tableName}` ( + `user_id` LARGEINT NOT NULL COMMENT '\"用户id\"', + `num` SMALLINT SUM NOT NULL COMMENT '\"数量\"' + ) ENGINE=OLAP + AGGREGATE KEY(`user_id`) + COMMENT 'OLAP' + DISTRIBUTED BY HASH(`user_id`) BUCKETS 2 + PROPERTIES ('replication_num' = '1') ; + """ + sql """ + insert into ${tableName} values (1,1),(1,2); + """ + + sql """ + CREATE MATERIALIZED VIEW ${mvName} + BUILD DEFERRED REFRESH AUTO ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS + SELECT * FROM ${tableName}; + """ + + sql """ + REFRESH MATERIALIZED VIEW ${mvName}; + """ + + def jobName = getJobName(dbName, mvName); + log.info(jobName) + waitingMTMVTaskFinished(jobName) + + order_qt_select "SELECT * FROM ${mvName}" + + sql """drop table if exists `${tableName}`""" + sql """drop materialized view if exists ${mvName};""" +} diff --git a/regression-test/suites/mtmv_p0/test_mor_table_mtmv.groovy b/regression-test/suites/mtmv_p0/test_mor_table_mtmv.groovy new file mode 100644 index 00000000000..7795745c507 --- /dev/null +++ b/regression-test/suites/mtmv_p0/test_mor_table_mtmv.groovy @@ -0,0 +1,62 @@ +// 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. + +import org.junit.Assert; + +suite("test_mor_table_mtmv") { + def tableName = "t_test_mor_table_mtmv_user" + def mvName = "test_mor_table_mtmv" + def dbName = "regression_test_mtmv_p0" + sql """drop table if exists `${tableName}`""" + sql """drop materialized view if exists ${mvName};""" + + sql """ + CREATE TABLE `${tableName}` ( + `user_id` LARGEINT NOT NULL COMMENT '\"用户id\"', + `num` SMALLINT NOT NULL COMMENT '\"数量\"' + ) ENGINE=OLAP + UNIQUE KEY(`user_id`) + COMMENT 'OLAP' + DISTRIBUTED BY HASH(`user_id`) BUCKETS 2 + PROPERTIES ('replication_num' = '1') ; + """ + sql """ + insert into ${tableName} values (1,1),(1,2); + """ + + sql """ + CREATE MATERIALIZED VIEW ${mvName} + BUILD DEFERRED REFRESH AUTO ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS + SELECT * FROM ${tableName}; + """ + + sql """ + REFRESH MATERIALIZED VIEW ${mvName}; + """ + + def jobName = getJobName(dbName, mvName); + log.info(jobName) + waitingMTMVTaskFinished(jobName) + + order_qt_select "SELECT * FROM ${mvName}" + + sql """drop table if exists `${tableName}`""" + sql """drop materialized view if exists ${mvName};""" +} diff --git a/regression-test/suites/mtmv_p0/test_mow_table_mtmv.groovy b/regression-test/suites/mtmv_p0/test_mow_table_mtmv.groovy new file mode 100644 index 00000000000..954b03c402a --- /dev/null +++ b/regression-test/suites/mtmv_p0/test_mow_table_mtmv.groovy @@ -0,0 +1,62 @@ +// 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. + +import org.junit.Assert; + +suite("test_mow_table_mtmv") { + def tableName = "t_test_mow_table_mtmv_user" + def mvName = "test_mow_table_mtmv" + def dbName = "regression_test_mtmv_p0" + sql """drop table if exists `${tableName}`""" + sql """drop materialized view if exists ${mvName};""" + + sql """ + CREATE TABLE `${tableName}` ( + `user_id` LARGEINT NOT NULL COMMENT '\"用户id\"', + `num` SMALLINT NOT NULL COMMENT '\"数量\"' + ) ENGINE=OLAP + UNIQUE KEY(`user_id`) + COMMENT 'OLAP' + DISTRIBUTED BY HASH(`user_id`) BUCKETS 2 + PROPERTIES ('replication_num' = '1','enable_unique_key_merge_on_write' = 'true') ; + """ + sql """ + insert into ${tableName} values (1,1),(1,2); + """ + + sql """ + CREATE MATERIALIZED VIEW ${mvName} + BUILD DEFERRED REFRESH AUTO ON MANUAL + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS + SELECT * FROM ${tableName}; + """ + + sql """ + REFRESH MATERIALIZED VIEW ${mvName}; + """ + + def jobName = getJobName(dbName, mvName); + log.info(jobName) + waitingMTMVTaskFinished(jobName) + + order_qt_select "SELECT * FROM ${mvName}" + + sql """drop table if exists `${tableName}`""" + sql """drop materialized view if exists ${mvName};""" +} diff --git a/regression-test/suites/mtmv_p0/test_partition_limit_mtmv.groovy b/regression-test/suites/mtmv_p0/test_partition_limit_mtmv.groovy new file mode 100644 index 00000000000..c49a40b7e32 --- /dev/null +++ b/regression-test/suites/mtmv_p0/test_partition_limit_mtmv.groovy @@ -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. + +import org.junit.Assert; + +suite("test_partition_limit_mtmv") { + def tableNameNum = "t_test_partition_limit_mtmv_user_num" + def mvName = "test_partition_limit_mtmv" + def dbName = "regression_test_mtmv_p0" + sql """drop table if exists `${tableNameNum}`""" + sql """drop materialized view if exists ${mvName};""" + + sql """ + CREATE TABLE `${tableNameNum}` ( + `user_id` LARGEINT NOT NULL COMMENT '\"用户id\"', + `date` DATE NOT NULL COMMENT '\"数据灌入日期时间\"', + `num` SMALLINT NOT NULL COMMENT '\"数量\"' + ) ENGINE=OLAP + DUPLICATE KEY(`user_id`, `date`, `num`) + COMMENT 'OLAP' + PARTITION BY RANGE(`date`) + (PARTITION p201701_1000 VALUES [('0000-01-01'), ('2017-02-01')), + PARTITION p201702_2000 VALUES [('2017-02-01'), ('2017-03-01')), + PARTITION p201703_all VALUES [('2017-03-01'), ('2017-04-01'))) + DISTRIBUTED BY HASH(`user_id`) BUCKETS 2 + PROPERTIES ('replication_num' = '1') ; + """ + sql """ set create_table_partition_max_num=1;""" + try { + sql """ + CREATE MATERIALIZED VIEW ${mvName} + BUILD DEFERRED REFRESH AUTO ON MANUAL + partition by(`date`) + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS + SELECT * FROM ${tableNameNum}; + """ + Assert.fail(); + } catch (Exception e) { + log.info(e.getMessage()) + } + sql """ set create_table_partition_max_num=10;""" + try { + sql """ + CREATE MATERIALIZED VIEW ${mvName} + BUILD DEFERRED REFRESH AUTO ON MANUAL + partition by(`date`) + DISTRIBUTED BY RANDOM BUCKETS 2 + PROPERTIES ('replication_num' = '1') + AS + SELECT * FROM ${tableNameNum}; + """ + } catch (Exception e) { + log.info(e.getMessage()) + Assert.fail(); + } + sql """drop table if exists `${tableNameNum}`""" + sql """drop materialized view if exists ${mvName};""" +} diff --git a/regression-test/suites/mtmv_p0/test_partition_refresh_mtmv.groovy b/regression-test/suites/mtmv_p0/test_partition_refresh_mtmv.groovy index 1863be6b875..c9a3fd27128 100644 --- a/regression-test/suites/mtmv_p0/test_partition_refresh_mtmv.groovy +++ b/regression-test/suites/mtmv_p0/test_partition_refresh_mtmv.groovy @@ -15,6 +15,8 @@ // specific language governing permissions and limitations // under the License. +import org.junit.Assert; + suite("test_partition_refresh_mtmv") { def tableNameNum = "t_test_pr_mtmv_user_num" def tableNameUser = "t_test_pr_mtmv_user" --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org