This is an automated email from the ASF dual-hosted git repository. dataroaring pushed a commit to branch branch-3.0 in repository https://gitbox.apache.org/repos/asf/doris.git
commit 5aeef69aa70eb05da5208dbac3b2f48509d85833 Author: morrySnow <101034200+morrys...@users.noreply.github.com> AuthorDate: Mon Sep 9 12:19:56 2024 +0800 [opt](Nereids) fix several insert into related issues (#40467) - http_stream TVF should always generate one fragment plan - http_stream TVF plan should not check root as scan node - distinguish group_commit TVF with normal insert statement - index and generate slot should based on type cast base slot - agg_state could cast from nullable to non-nullable - colocated and bucket scan range compute should only on scan node --- .../main/java/org/apache/doris/catalog/Type.java | 4 - .../doris/nereids/parser/LogicalPlanBuilder.java | 2 +- .../doris/nereids/rules/analysis/BindSink.java | 81 +++++++++++----- .../expressions/functions/table/HttpStream.java | 11 +++ .../trees/plans/commands/info/DMLCommandType.java | 2 + .../main/java/org/apache/doris/qe/Coordinator.java | 5 +- .../java/org/apache/doris/qe/StmtExecutor.java | 17 ++-- .../insert_into_table/insert_use_table_id.out | 48 --------- .../agg_state/max/test_agg_state_max.groovy | 2 +- .../insert_group_commit_into_unique.groovy | 9 +- .../insert_into_table/insert_use_table_id.groovy | 107 --------------------- 11 files changed, 90 insertions(+), 198 deletions(-) diff --git a/fe/fe-common/src/main/java/org/apache/doris/catalog/Type.java b/fe/fe-common/src/main/java/org/apache/doris/catalog/Type.java index 9d2abc75d44..5a2b3cf8fc5 100644 --- a/fe/fe-common/src/main/java/org/apache/doris/catalog/Type.java +++ b/fe/fe-common/src/main/java/org/apache/doris/catalog/Type.java @@ -868,10 +868,6 @@ public abstract class Type { return false; } for (int i = 0; i < sourceAggState.getSubTypes().size(); i++) { - // target subtype is not null but source subtype is nullable - if (!targetAggState.getSubTypeNullables().get(i) && sourceAggState.getSubTypeNullables().get(i)) { - return false; - } if (!canCastTo(sourceAggState.getSubTypes().get(i), targetAggState.getSubTypes().get(i))) { return false; } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java index 061a58a634d..28291a88518 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/parser/LogicalPlanBuilder.java @@ -589,7 +589,7 @@ public class LogicalPlanBuilder extends DorisParserBaseVisitor<Object> { isAutoDetect, isOverwrite, ConnectContext.get().getSessionVariable().isEnableUniqueKeyPartialUpdate(), - DMLCommandType.INSERT, + ctx.tableId == null ? DMLCommandType.INSERT : DMLCommandType.GROUP_COMMIT, plan); Optional<LogicalPlan> cte = Optional.empty(); if (ctx.cte() != null) { diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSink.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSink.java index c9e7f07f5d0..6d8ad94242b 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSink.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSink.java @@ -68,6 +68,7 @@ import org.apache.doris.nereids.trees.plans.visitor.InferPlanOutputAlias; import org.apache.doris.nereids.types.DataType; import org.apache.doris.nereids.types.StringType; import org.apache.doris.nereids.types.coercion.CharacterType; +import org.apache.doris.nereids.util.ExpressionUtils; import org.apache.doris.nereids.util.RelationUtil; import org.apache.doris.nereids.util.TypeCoercionUtils; @@ -126,7 +127,8 @@ public class BindSink implements AnalysisRuleFactory { && table.getSequenceMapCol() != null && sink.getColNames().contains(table.getSequenceMapCol()); Pair<List<Column>, Integer> bindColumnsResult = - bindTargetColumns(table, sink.getColNames(), childHasSeqCol, needExtraSeqCol); + bindTargetColumns(table, sink.getColNames(), childHasSeqCol, needExtraSeqCol, + sink.getDMLCommandType() == DMLCommandType.GROUP_COMMIT); List<Column> bindColumns = bindColumnsResult.first; int extraColumnsNum = bindColumnsResult.second; @@ -176,8 +178,12 @@ public class BindSink implements AnalysisRuleFactory { .filter(col -> col.getName().equalsIgnoreCase(table.getSequenceMapCol())) .findFirst(); } else { - if (!sink.getColNames().isEmpty()) { - if (sink.getColNames().stream() + // ATTN: must use bindColumns here. Because of insert into from group_commit tvf submitted by BE + // do not follow any column list with target table, but it contains all inviable data in sink's + // child. THis is different with other insert action that contain non-inviable data by default. + if (!bindColumns.isEmpty()) { + if (bindColumns.stream() + .map(Column::getName) .anyMatch(c -> c.equalsIgnoreCase(Column.SEQUENCE_COL))) { haveInputSeqCol = true; // case2.a } // else case2.b @@ -205,7 +211,8 @@ public class BindSink implements AnalysisRuleFactory { Map<String, NamedExpression> columnToOutput = getColumnToOutput( ctx, table, isPartialUpdate, boundSink, child); - LogicalProject<?> fullOutputProject = getOutputProjectByCoercion(table.getFullSchema(), child, columnToOutput); + LogicalProject<?> fullOutputProject = getOutputProjectByCoercion( + table.getFullSchema(), child, columnToOutput); return boundSink.withChildAndUpdateOutput(fullOutputProject); } @@ -267,15 +274,14 @@ public class BindSink implements AnalysisRuleFactory { // we need to insert all the columns of the target table // although some columns are not mentions. // so we add a projects to supply the default value. - Map<Column, NamedExpression> columnToChildOutput = Maps.newHashMap(); for (int i = 0; i < child.getOutput().size(); ++i) { columnToChildOutput.put(boundSink.getCols().get(i), child.getOutput().get(i)); } - Map<String, NamedExpression> columnToOutput = Maps.newTreeMap(String.CASE_INSENSITIVE_ORDER); + Map<String, NamedExpression> columnToReplaced = Maps.newTreeMap(String.CASE_INSENSITIVE_ORDER); + Map<Expression, Expression> replaceMap = Maps.newHashMap(); NereidsParser expressionParser = new NereidsParser(); - List<Column> generatedColumns = Lists.newArrayList(); List<Column> materializedViewColumn = Lists.newArrayList(); // generate slots not mentioned in sql, mv slots and shaded slots. @@ -291,7 +297,12 @@ public class BindSink implements AnalysisRuleFactory { // do not process explicitly use DEFAULT value here: // insert into table t values(DEFAULT) && !(columnToChildOutput.get(column) instanceof DefaultValueSlot)) { - columnToOutput.put(column.getName(), columnToChildOutput.get(column)); + Alias output = new Alias(TypeCoercionUtils.castIfNotSameType( + columnToChildOutput.get(column), DataType.fromCatalogType(column.getType())), + column.getName()); + columnToOutput.put(column.getName(), output); + columnToReplaced.put(column.getName(), output.toSlot()); + replaceMap.put(output.toSlot(), output.child()); } else { if (table instanceof OlapTable && ((OlapTable) table).hasSequenceCol() && column.getName().equals(Column.SEQUENCE_COL) @@ -312,6 +323,8 @@ public class BindSink implements AnalysisRuleFactory { seqColumn = new Alias(seqColumn, column.getName()); } columnToOutput.put(column.getName(), seqColumn); + columnToReplaced.put(column.getName(), seqColumn.toSlot()); + replaceMap.put(seqColumn.toSlot(), seqColumn.child(0)); } } else if (isPartialUpdate) { // If the current load is a partial update, the values of unmentioned @@ -328,9 +341,12 @@ public class BindSink implements AnalysisRuleFactory { Expression defualtValueExpression = ExpressionAnalyzer.analyzeFunction( boundSink, ctx.cascadesContext, unboundFunctionDefaultValue ); - columnToOutput.put(column.getName(), - new Alias(defualtValueExpression, column.getName()) - ); + Alias output = new Alias(TypeCoercionUtils.castIfNotSameType( + defualtValueExpression, DataType.fromCatalogType(column.getType())), + column.getName()); + columnToOutput.put(column.getName(), output); + columnToReplaced.put(column.getName(), output.toSlot()); + replaceMap.put(output.toSlot(), output.child()); } else { continue; } @@ -343,10 +359,11 @@ public class BindSink implements AnalysisRuleFactory { } // Otherwise, the unmentioned columns should be filled with default values // or null values - columnToOutput.put(column.getName(), new Alias( - new NullLiteral(DataType.fromCatalogType(column.getType())), - column.getName() - )); + Alias output = new Alias(new NullLiteral(DataType.fromCatalogType(column.getType())), + column.getName()); + columnToOutput.put(column.getName(), output); + columnToReplaced.put(column.getName(), output.toSlot()); + replaceMap.put(output.toSlot(), output.child()); } else { try { // it comes from the original planner, if default value expression is @@ -365,8 +382,12 @@ public class BindSink implements AnalysisRuleFactory { if (defualtValueExpression instanceof Alias) { defualtValueExpression = ((Alias) defualtValueExpression).child(); } - columnToOutput.put(column.getName(), - new Alias(defualtValueExpression, column.getName())); + Alias output = new Alias((TypeCoercionUtils.castIfNotSameType( + defualtValueExpression, DataType.fromCatalogType(column.getType()))), + column.getName()); + columnToOutput.put(column.getName(), output); + columnToReplaced.put(column.getName(), output.toSlot()); + replaceMap.put(output.toSlot(), output.child()); } } catch (Exception e) { throw new AnalysisException(e.getMessage(), e.getCause()); @@ -380,13 +401,16 @@ public class BindSink implements AnalysisRuleFactory { for (Column column : generatedColumns) { GeneratedColumnInfo info = column.getGeneratedColumnInfo(); Expression parsedExpression = new NereidsParser().parseExpression(info.getExpr().toSqlWithoutTbl()); - Expression boundExpression = new CustomExpressionAnalyzer(boundSink, ctx.cascadesContext, columnToOutput) + Expression boundExpression = new CustomExpressionAnalyzer(boundSink, ctx.cascadesContext, columnToReplaced) .analyze(parsedExpression); if (boundExpression instanceof Alias) { boundExpression = ((Alias) boundExpression).child(); } - NamedExpression slot = new Alias(boundExpression, info.getExprSql()); - columnToOutput.put(column.getName(), slot); + boundExpression = ExpressionUtils.replace(boundExpression, replaceMap); + Alias output = new Alias(boundExpression, info.getExprSql()); + columnToOutput.put(column.getName(), output); + columnToReplaced.put(column.getName(), output.toSlot()); + replaceMap.put(output.toSlot(), output.child()); } for (Column column : materializedViewColumn) { if (column.isMaterializedViewColumn()) { @@ -400,12 +424,15 @@ public class BindSink implements AnalysisRuleFactory { // may not be bound, we have to bind it again. // for example: to_bitmap. Expression boundExpression = new CustomExpressionAnalyzer( - boundSink, ctx.cascadesContext, columnToOutput).analyze(parsedExpression); + boundSink, ctx.cascadesContext, columnToReplaced).analyze(parsedExpression); if (boundExpression instanceof Alias) { boundExpression = ((Alias) boundExpression).child(); } - NamedExpression slot = new Alias(boundExpression, column.getDefineExpr().toSqlWithoutTbl()); - columnToOutput.put(column.getName(), slot); + boundExpression = ExpressionUtils.replace(boundExpression, replaceMap); + boundExpression = TypeCoercionUtils.castIfNotSameType(boundExpression, + DataType.fromCatalogType(column.getType())); + Alias output = new Alias(boundExpression, column.getDefineExpr().toSqlWithoutTbl()); + columnToOutput.put(column.getName(), output); } } return columnToOutput; @@ -554,12 +581,14 @@ public class BindSink implements AnalysisRuleFactory { } private Pair<List<Column>, Integer> bindTargetColumns(OlapTable table, List<String> colsName, - boolean childHasSeqCol, boolean needExtraSeqCol) { + boolean childHasSeqCol, boolean needExtraSeqCol, boolean isGroupCommit) { // if the table set sequence column in stream load phase, the sequence map column is null, we query it. if (colsName.isEmpty()) { + // ATTN: group commit without column list should return all base index column + // because it already prepares data for these columns. return Pair.of(table.getBaseSchema(true).stream() - .filter(c -> validColumn(c, childHasSeqCol)) - .collect(ImmutableList.toImmutableList()), 0); + .filter(c -> isGroupCommit || validColumn(c, childHasSeqCol)) + .collect(ImmutableList.toImmutableList()), 0); } else { int extraColumnsNum = (needExtraSeqCol ? 1 : 0); List<String> processedColsName = Lists.newArrayList(colsName); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/table/HttpStream.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/table/HttpStream.java index de052b078db..8e35e25240e 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/table/HttpStream.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/functions/table/HttpStream.java @@ -19,12 +19,17 @@ package org.apache.doris.nereids.trees.expressions.functions.table; import org.apache.doris.catalog.FunctionSignature; import org.apache.doris.nereids.exceptions.AnalysisException; +import org.apache.doris.nereids.properties.DistributionSpecHash; +import org.apache.doris.nereids.properties.DistributionSpecHash.ShuffleType; +import org.apache.doris.nereids.properties.PhysicalProperties; import org.apache.doris.nereids.trees.expressions.Properties; import org.apache.doris.nereids.trees.expressions.visitor.ExpressionVisitor; import org.apache.doris.nereids.types.coercion.AnyDataType; import org.apache.doris.tablefunction.HttpStreamTableValuedFunction; import org.apache.doris.tablefunction.TableValuedFunctionIf; +import com.google.common.collect.ImmutableList; + import java.util.Map; /** http_stream */ @@ -49,6 +54,12 @@ public class HttpStream extends TableValuedFunction { } } + @Override + public PhysicalProperties getPhysicalProperties() { + return PhysicalProperties.createHash(new DistributionSpecHash(ImmutableList.of(), + ShuffleType.EXECUTION_BUCKETED)); + } + @Override public <R, C> R accept(ExpressionVisitor<R, C> visitor, C context) { return visitor.visitHttpStream(this, context); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/DMLCommandType.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/DMLCommandType.java index 18d8179abe4..aa97f26df18 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/DMLCommandType.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/info/DMLCommandType.java @@ -27,6 +27,8 @@ public enum DMLCommandType { NONE, // for INSERT INTO or INSERT INTO SELECT INSERT, + // for group_commit tvf + GROUP_COMMIT, // for UPDATE UPDATE, // for DELETE diff --git a/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java b/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java index 701926a0889..98eca5c0a88 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java +++ b/fe/fe-core/src/main/java/org/apache/doris/qe/Coordinator.java @@ -2160,9 +2160,10 @@ public class Coordinator implements CoordInterface { FragmentScanRangeAssignment assignment = fragmentExecParamsMap.get(scanNode.getFragmentId()).scanRangeAssignment; boolean fragmentContainsColocateJoin = isColocateFragment(scanNode.getFragment(), - scanNode.getFragment().getPlanRoot()); + scanNode.getFragment().getPlanRoot()) && (scanNode instanceof OlapScanNode); boolean fragmentContainsBucketShuffleJoin = bucketShuffleJoinController - .isBucketShuffleJoin(scanNode.getFragmentId().asInt(), scanNode.getFragment().getPlanRoot()); + .isBucketShuffleJoin(scanNode.getFragmentId().asInt(), scanNode.getFragment().getPlanRoot()) + && (scanNode instanceof OlapScanNode); // A fragment may contain both colocate join and bucket shuffle join // on need both compute scanRange to init basic data for query coordinator diff --git a/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java b/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java index a0eb8b55a0c..bd16d011a59 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java +++ b/fe/fe-core/src/main/java/org/apache/doris/qe/StmtExecutor.java @@ -3433,8 +3433,16 @@ public class StmtExecutor { httpStreamParams.setLabel(insertExecutor.getLabelName()); PlanNode planRoot = planner.getFragments().get(0).getPlanRoot(); - Preconditions.checkState(planRoot instanceof TVFScanNode || planRoot instanceof GroupCommitScanNode, - "Nereids' planNode cannot be converted to " + planRoot.getClass().getName()); + boolean isValidPlan = !planner.getScanNodes().isEmpty(); + for (ScanNode scanNode : planner.getScanNodes()) { + if (!(scanNode instanceof TVFScanNode || planRoot instanceof GroupCommitScanNode)) { + isValidPlan = false; + break; + } + } + if (!isValidPlan) { + throw new AnalysisException("plan is invalid: " + planRoot.getExplainString()); + } } catch (QueryStateException e) { LOG.debug("Command(" + originStmt.originStmt + ") process failed.", e); context.setState(e.getQueryState()); @@ -3505,11 +3513,8 @@ public class StmtExecutor { LOG.warn("Analyze failed. {}", context.getQueryIdentifier(), e); throw ((NereidsException) e).getException(); } - boolean isInsertIntoCommand = parsedStmt != null && parsedStmt instanceof LogicalPlanAdapter - && ((LogicalPlanAdapter) parsedStmt).getLogicalPlan() instanceof InsertIntoTableCommand; if (e instanceof NereidsException - && !context.getSessionVariable().enableFallbackToOriginalPlanner - && !isInsertIntoCommand) { + && !context.getSessionVariable().enableFallbackToOriginalPlanner) { LOG.warn("Analyze failed. {}", context.getQueryIdentifier(), e); throw ((NereidsException) e).getException(); } diff --git a/regression-test/data/nereids_p0/insert_into_table/insert_use_table_id.out b/regression-test/data/nereids_p0/insert_into_table/insert_use_table_id.out deleted file mode 100644 index d0020443bf6..00000000000 --- a/regression-test/data/nereids_p0/insert_into_table/insert_use_table_id.out +++ /dev/null @@ -1,48 +0,0 @@ --- This file is automatically generated. You should know what you did if you want to edit this --- !sql_cross_join -- -1 10 1 1 1.0 2000-01-01 1 10 10 10.0 2000-01-10 1 -1 10 1 1 1.0 2000-01-01 1 10 10 10.0 2000-01-10 4 -1 10 1 1 1.0 2000-01-01 1 10 10 10.0 2000-01-10 5 -1 10 1 1 1.0 2000-01-01 2 20 20 20.0 2000-01-20 1 -1 10 1 1 1.0 2000-01-01 2 20 20 20.0 2000-01-20 4 -1 10 1 1 1.0 2000-01-01 2 20 20 20.0 2000-01-20 5 -1 10 1 1 1.0 2000-01-01 3 30 30 30.0 2000-01-30 1 -1 10 1 1 1.0 2000-01-01 3 30 30 30.0 2000-01-30 4 -1 10 1 1 1.0 2000-01-01 3 30 30 30.0 2000-01-30 5 -1 10 1 1 1.0 2000-01-01 4 4 4 4.0 2000-01-04 1 -1 10 1 1 1.0 2000-01-01 4 4 4 4.0 2000-01-04 4 -1 10 1 1 1.0 2000-01-01 4 4 4 4.0 2000-01-04 5 -1 10 1 1 1.0 2000-01-01 5 5 5 5.0 2000-01-05 1 -1 10 1 1 1.0 2000-01-01 5 5 5 5.0 2000-01-05 4 -1 10 1 1 1.0 2000-01-01 5 5 5 5.0 2000-01-05 5 -2 20 2 2 2.0 2000-01-02 1 10 10 10.0 2000-01-10 1 -2 20 2 2 2.0 2000-01-02 1 10 10 10.0 2000-01-10 4 -2 20 2 2 2.0 2000-01-02 1 10 10 10.0 2000-01-10 5 -2 20 2 2 2.0 2000-01-02 2 20 20 20.0 2000-01-20 1 -2 20 2 2 2.0 2000-01-02 2 20 20 20.0 2000-01-20 4 -2 20 2 2 2.0 2000-01-02 2 20 20 20.0 2000-01-20 5 -2 20 2 2 2.0 2000-01-02 3 30 30 30.0 2000-01-30 1 -2 20 2 2 2.0 2000-01-02 3 30 30 30.0 2000-01-30 4 -2 20 2 2 2.0 2000-01-02 3 30 30 30.0 2000-01-30 5 -2 20 2 2 2.0 2000-01-02 4 4 4 4.0 2000-01-04 1 -2 20 2 2 2.0 2000-01-02 4 4 4 4.0 2000-01-04 4 -2 20 2 2 2.0 2000-01-02 4 4 4 4.0 2000-01-04 5 -2 20 2 2 2.0 2000-01-02 5 5 5 5.0 2000-01-05 1 -2 20 2 2 2.0 2000-01-02 5 5 5 5.0 2000-01-05 4 -2 20 2 2 2.0 2000-01-02 5 5 5 5.0 2000-01-05 5 -3 30 3 3 3.0 2000-01-03 1 10 10 10.0 2000-01-10 1 -3 30 3 3 3.0 2000-01-03 1 10 10 10.0 2000-01-10 4 -3 30 3 3 3.0 2000-01-03 1 10 10 10.0 2000-01-10 5 -3 30 3 3 3.0 2000-01-03 2 20 20 20.0 2000-01-20 1 -3 30 3 3 3.0 2000-01-03 2 20 20 20.0 2000-01-20 4 -3 30 3 3 3.0 2000-01-03 2 20 20 20.0 2000-01-20 5 -3 30 3 3 3.0 2000-01-03 3 30 30 30.0 2000-01-30 1 -3 30 3 3 3.0 2000-01-03 3 30 30 30.0 2000-01-30 4 -3 30 3 3 3.0 2000-01-03 3 30 30 30.0 2000-01-30 5 -3 30 3 3 3.0 2000-01-03 4 4 4 4.0 2000-01-04 1 -3 30 3 3 3.0 2000-01-03 4 4 4 4.0 2000-01-04 4 -3 30 3 3 3.0 2000-01-03 4 4 4 4.0 2000-01-04 5 -3 30 3 3 3.0 2000-01-03 5 5 5 5.0 2000-01-05 1 -3 30 3 3 3.0 2000-01-03 5 5 5 5.0 2000-01-05 4 -3 30 3 3 3.0 2000-01-03 5 5 5 5.0 2000-01-05 5 - diff --git a/regression-test/suites/datatype_p0/agg_state/max/test_agg_state_max.groovy b/regression-test/suites/datatype_p0/agg_state/max/test_agg_state_max.groovy index 983f51beed1..a71da554afb 100644 --- a/regression-test/suites/datatype_p0/agg_state/max/test_agg_state_max.groovy +++ b/regression-test/suites/datatype_p0/agg_state/max/test_agg_state_max.groovy @@ -30,7 +30,7 @@ suite("test_agg_state_max") { test { sql "insert into a_table values(100,max_state(null));" - exception "can not cast from origin type agg_state" + exception "illegal for non_nullable" } sql """insert into a_table diff --git a/regression-test/suites/insert_p0/insert_group_commit_into_unique.groovy b/regression-test/suites/insert_p0/insert_group_commit_into_unique.groovy index ca280cd17d8..8ae0d41565d 100644 --- a/regression-test/suites/insert_p0/insert_group_commit_into_unique.groovy +++ b/regression-test/suites/insert_p0/insert_group_commit_into_unique.groovy @@ -86,7 +86,8 @@ suite("insert_group_commit_into_unique") { UNIQUE KEY(`id`, `name`) DISTRIBUTED BY HASH(`id`) BUCKETS 1 PROPERTIES ( - "replication_num" = "1" + "replication_num" = "1", + "group_commit_interval_ms" = "100" ); """ @@ -171,7 +172,8 @@ suite("insert_group_commit_into_unique") { DISTRIBUTED BY HASH(`id`) BUCKETS 1 PROPERTIES ( "replication_num" = "1", - "function_column.sequence_col" = "score" + "function_column.sequence_col" = "score", + "group_commit_interval_ms" = "100" ); """ @@ -257,7 +259,8 @@ suite("insert_group_commit_into_unique") { DISTRIBUTED BY HASH(`id`) BUCKETS 1 PROPERTIES ( "replication_num" = "1", - "function_column.sequence_type" = "int" + "function_column.sequence_type" = "int", + "group_commit_interval_ms" = "100" ); """ diff --git a/regression-test/suites/nereids_p0/insert_into_table/insert_use_table_id.groovy b/regression-test/suites/nereids_p0/insert_into_table/insert_use_table_id.groovy deleted file mode 100644 index 930fe35b60e..00000000000 --- a/regression-test/suites/nereids_p0/insert_into_table/insert_use_table_id.groovy +++ /dev/null @@ -1,107 +0,0 @@ -// 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('nereids_insert_use_table_id') { - sql 'set enable_nereids_planner=true' - sql 'set enable_fallback_to_original_planner=false' - sql 'set enable_nereids_dml=true' - sql 'set enable_strict_consistency_dml=true' - - // sql 'CREATE DATABASE IF NOT EXISTS dnereids_insert_use_table_id_test' - // sql 'use nereids_insert_use_table_id_test' - - def t1 = 'table_id_value_t1' - def t2 = 'table_id_value_t2' - def t3 = 'table_id_value_t3' - - sql "drop table if exists ${t1}" - sql "drop table if exists ${t2}" - sql "drop table if exists ${t3}" - - sql """ - create table ${t1} ( - id int, - id1 int, - c1 bigint, - c2 string, - c3 double, - c4 date - ) unique key (id, id1) - distributed by hash(id, id1) buckets 13 - properties( - 'replication_num'='1', - "function_column.sequence_col" = "c4" - ); - """ - - sql """ - create table ${t2} ( - id int, - c1 bigint, - c2 string, - c3 double, - c4 date - ) unique key (id) - distributed by hash(id) buckets 13 - properties( - 'replication_num'='1' - ); - """ - - sql """ - create table ${t3} ( - id int - ) distributed by hash(id) buckets 13 - properties( - 'replication_num'='1' - ); - """ - - - sql """ - INSERT INTO DORIS_INTERNAL_TABLE_ID(${getTableId(t1)}) VALUES - (1, (1 + 9) * (10 - 9), 1, '1', 1.0, '2000-01-01'), - (2, 20, 2, '2', 2.0, days_add('2000-01-01', 1)), - (3, 30, 3, '3', 3.0, makedate(2000, 3)); - """ - - sql """ - INSERT INTO DORIS_INTERNAL_TABLE_ID(${getTableId(t2)}) VALUES - (1, 10, '10', 10.0, '2000-01-10'), - (2, 20, '20', 20.0, '2000-01-20'), - (3, 30, '30', 30.0, '2000-01-30'), - (4, 4, '4', 4.0, '2000-01-04'), - (5, 5, '5', 5.0, '2000-01-05'); - """ - - sql """ - INSERT INTO DORIS_INTERNAL_TABLE_ID(${getTableId(t3)}) VALUES - (1), - (4), - (5); - """ - - sql "sync" - qt_sql_cross_join "select * from ${t1}, ${t2}, ${t3} order by ${t1}.id, ${t1}.id1, ${t2}.id, ${t3}.id" - - -} - - --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org