This is an automated email from the ASF dual-hosted git repository. starocean999 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 ba4c738ac7 [Feature](Nereids) support values table (#23121) ba4c738ac7 is described below commit ba4c738ac744aed77543f446becb15c4dce8bb18 Author: mch_ucchi <41606806+sohardforan...@users.noreply.github.com> AuthorDate: Fri Sep 15 21:46:37 2023 +0800 [Feature](Nereids) support values table (#23121) support insert into table values(...) for Nereids. sql like: insert into t values(1, 2, 3) insert into t values(1 + 1, dayofweek(now()), 4), (4, 5, 6) insert into t values('1', '6.5', cast(1.5 as int)) --- .../antlr4/org/apache/doris/nereids/DorisParser.g4 | 10 +- .../doris/nereids/parser/LogicalPlanBuilder.java | 24 +++++ .../doris/nereids/rules/analysis/BindSink.java | 40 +++++-- .../plans/commands/InsertIntoTableCommand.java | 2 +- .../trees/plans/visitor/CommandVisitor.java | 2 +- .../java/org/apache/doris/qe/StmtExecutor.java | 3 + .../trees/plans/ExplainInsertCommandTest.java | 14 ++- .../nereids_p0/insert_into_table/insert_values.out | 51 +++++++++ .../insert_into_table/complex_insert.groovy | 1 - .../insert_into_table/insert_values.groovy | 118 +++++++++++++++++++++ 10 files changed, 253 insertions(+), 12 deletions(-) diff --git a/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 b/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 index d52473cd0c..a83a9dc30c 100644 --- a/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 +++ b/fe/fe-core/src/main/antlr4/org/apache/doris/nereids/DorisParser.g4 @@ -145,6 +145,7 @@ setQuantifier queryPrimary : querySpecification #queryPrimaryDefault | LEFT_PAREN query RIGHT_PAREN #subquery + | inlineTable #valuesTable ; querySpecification @@ -396,6 +397,11 @@ aggTypeDef tabletList : TABLET LEFT_PAREN tabletIdList+=INTEGER_VALUE (COMMA tabletIdList+=INTEGER_VALUE)* RIGHT_PAREN ; + + +inlineTable + : VALUES rowConstructor (COMMA rowConstructor)* + ; // -----------------Expression----------------- namedExpression @@ -430,7 +436,9 @@ booleanExpression | left=booleanExpression operator=DOUBLEPIPES right=booleanExpression #doublePipes ; - +rowConstructor + : LEFT_PAREN namedExpression (COMMA namedExpression)+ RIGHT_PAREN + ; predicate : NOT? kind=BETWEEN lower=valueExpression AND upper=valueExpression 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 23675cc56b..19a9a244be 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 @@ -79,6 +79,7 @@ import org.apache.doris.nereids.DorisParser.IdentifierSeqContext; import org.apache.doris.nereids.DorisParser.InPartitionDefContext; import org.apache.doris.nereids.DorisParser.IndexDefContext; import org.apache.doris.nereids.DorisParser.IndexDefsContext; +import org.apache.doris.nereids.DorisParser.InlineTableContext; import org.apache.doris.nereids.DorisParser.InsertIntoQueryContext; import org.apache.doris.nereids.DorisParser.IntegerLiteralContext; import org.apache.doris.nereids.DorisParser.IntervalContext; @@ -119,6 +120,7 @@ import org.apache.doris.nereids.DorisParser.RegularQuerySpecificationContext; import org.apache.doris.nereids.DorisParser.RelationContext; import org.apache.doris.nereids.DorisParser.RollupDefContext; import org.apache.doris.nereids.DorisParser.RollupDefsContext; +import org.apache.doris.nereids.DorisParser.RowConstructorContext; import org.apache.doris.nereids.DorisParser.SelectClauseContext; import org.apache.doris.nereids.DorisParser.SelectColumnClauseContext; import org.apache.doris.nereids.DorisParser.SelectHintContext; @@ -170,6 +172,7 @@ import org.apache.doris.nereids.properties.SelectHint; import org.apache.doris.nereids.properties.SelectHintLeading; import org.apache.doris.nereids.properties.SelectHintSetVar; import org.apache.doris.nereids.trees.expressions.Add; +import org.apache.doris.nereids.trees.expressions.Alias; import org.apache.doris.nereids.trees.expressions.And; import org.apache.doris.nereids.trees.expressions.BitAnd; import org.apache.doris.nereids.trees.expressions.BitNot; @@ -731,6 +734,15 @@ public class LogicalPlanBuilder extends DorisParserBaseVisitor<Object> { }); } + @Override + public LogicalPlan visitInlineTable(InlineTableContext ctx) { + List<LogicalPlan> exprsList = ctx.rowConstructor().stream() + .map(this::visitRowConstructor) + .map(LogicalPlan.class::cast) + .collect(ImmutableList.toImmutableList()); + return reduceToLogicalPlanTree(0, exprsList.size() - 1, exprsList, Qualifier.ALL); + } + /** * Create an aliased table reference. This is typically used in FROM clauses. */ @@ -1674,6 +1686,18 @@ public class LogicalPlanBuilder extends DorisParserBaseVisitor<Object> { return getExpression(ctx.expression()); } + @Override + public UnboundOneRowRelation visitRowConstructor(RowConstructorContext ctx) { + return new UnboundOneRowRelation( + StatementScopeIdGenerator.newRelationId(), + ctx.namedExpression().stream() + .map(this::visitNamedExpression) + .map(e -> (e instanceof NamedExpression) + ? ((NamedExpression) e) + : new Alias(e, e.toSql())) + .collect(ImmutableList.toImmutableList())); + } + @Override public List<Expression> visitNamedExpressionSeq(NamedExpressionSeqContext namedCtx) { return visit(namedCtx.namedExpression(), Expression.class); 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 790d226e02..459723e466 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 @@ -32,6 +32,8 @@ import org.apache.doris.nereids.exceptions.AnalysisException; import org.apache.doris.nereids.parser.NereidsParser; import org.apache.doris.nereids.rules.Rule; import org.apache.doris.nereids.rules.RuleType; +import org.apache.doris.nereids.rules.expression.ExpressionRewriteContext; +import org.apache.doris.nereids.rules.expression.rules.FunctionBinder; import org.apache.doris.nereids.trees.expressions.Alias; import org.apache.doris.nereids.trees.expressions.Expression; import org.apache.doris.nereids.trees.expressions.NamedExpression; @@ -106,8 +108,8 @@ public class BindSink implements AnalysisRuleFactory { if (ConnectContext.get() != null) { ConnectContext.get().getState().setIsQuery(true); } - // generate slots not mentioned in sql, mv slots and shaded slots. try { + // generate slots not mentioned in sql, mv slots and shaded slots. for (Column column : boundSink.getTargetTable().getFullSchema()) { if (column.isMaterializedViewColumn()) { List<SlotRef> refs = column.getRefColumns(); @@ -116,8 +118,13 @@ public class BindSink implements AnalysisRuleFactory { "mv column's ref column cannot be null"); Expression parsedExpression = expressionParser.parseExpression( column.getDefineExpr().toSql()); - Expression boundExpression = SlotReplacer.INSTANCE + Expression boundSlotExpression = SlotReplacer.INSTANCE .replace(parsedExpression, columnToOutput); + // the boundSlotExpression is an expression whose slots are bound but function + // may not be bound, we have to bind it again. + // for example: to_bitmap. + Expression boundExpression = FunctionBinder.INSTANCE.rewrite( + boundSlotExpression, new ExpressionRewriteContext(ctx.cascadesContext)); NamedExpression slot = boundExpression instanceof NamedExpression ? ((NamedExpression) boundExpression) @@ -140,10 +147,31 @@ public class BindSink implements AnalysisRuleFactory { column.getName() )); } else { - columnToOutput.put(column.getName(), - new Alias(Literal.of(column.getDefaultValue()) - .checkedCastTo(DataType.fromCatalogType(column.getType())), - column.getName())); + try { + // it comes from the original planner, if default value expression is + // null, we use the literal string of the default value, or it may be + // default value function, like CURRENT_TIMESTAMP. + if (column.getDefaultValueExpr() == null) { + columnToOutput.put(column.getName(), + new Alias(Literal.of(column.getDefaultValue()) + .checkedCastTo( + DataType.fromCatalogType(column.getType())), + column.getName())); + } else { + Expression defualtValueExpression = FunctionBinder.INSTANCE.rewrite( + new NereidsParser().parseExpression( + column.getDefaultValueExpr().toSql()), + new ExpressionRewriteContext(ctx.cascadesContext)); + NamedExpression slot = + defualtValueExpression instanceof NamedExpression + ? ((NamedExpression) defualtValueExpression) + : new Alias(defualtValueExpression); + + columnToOutput.put(column.getName(), slot); + } + } catch (Exception e) { + throw new AnalysisException(e.getMessage(), e.getCause()); + } } } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/InsertIntoTableCommand.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/InsertIntoTableCommand.java index 8f14f5efce..9563fd32ff 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/InsertIntoTableCommand.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/commands/InsertIntoTableCommand.java @@ -333,6 +333,6 @@ public class InsertIntoTableCommand extends Command implements ForwardWithSync, @Override public <R, C> R accept(PlanVisitor<R, C> visitor, C context) { - return visitor.visitInsertIntoCommand(this, context); + return visitor.visitInsertIntoTableCommand(this, context); } } diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java index 4c05d0b96e..e91e846e2f 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/visitor/CommandVisitor.java @@ -39,7 +39,7 @@ public interface CommandVisitor<R, C> { return visitCommand(createPolicy, context); } - default R visitInsertIntoCommand(InsertIntoTableCommand insertIntoSelectCommand, + default R visitInsertIntoTableCommand(InsertIntoTableCommand insertIntoSelectCommand, C context) { return visitCommand(insertIntoSelectCommand, context); } 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 0f7ff23972..c4ef5ad160 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 @@ -664,12 +664,15 @@ public class StmtExecutor { profile.getSummaryProfile().setQueryBeginTime(); context.setStmtId(STMT_ID_GENERATOR.incrementAndGet()); context.setQueryId(queryId); + // set isQuery first otherwise this state will be lost if some error occurs if (parsedStmt instanceof QueryStmt) { context.getState().setIsQuery(true); } try { + // parsedStmt maybe null here, we parse it. Or the predicate will not work. + parseByLegacy(); if (context.isTxnModel() && !(parsedStmt instanceof InsertStmt) && !(parsedStmt instanceof TransactionStmt)) { throw new TException("This is in a transaction, only insert, commit, rollback is acceptable."); diff --git a/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/ExplainInsertCommandTest.java b/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/ExplainInsertCommandTest.java index c36d764274..2d04cbda15 100644 --- a/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/ExplainInsertCommandTest.java +++ b/fe/fe-core/src/test/java/org/apache/doris/nereids/trees/plans/ExplainInsertCommandTest.java @@ -109,6 +109,16 @@ public class ExplainInsertCommandTest extends TestWithFeService { Assertions.assertEquals(4, getOutputFragment(sql).getOutputExprs().size()); } + @Test + public void testInsertIntoValues() throws Exception { + String sql = "explain insert into t1 values(1, 1, 1, 1), (2, 2, 2, 2), (3, 3, 3, 3)"; + Assertions.assertEquals(4, getOutputFragment(sql).getOutputExprs().size()); + sql = "explain insert into t2 values(1, 1, 1, 1), (2, 2, 2, 2), (3, 3, 3, 3)"; + Assertions.assertEquals(6, getOutputFragment(sql).getOutputExprs().size()); + sql = "explain insert into agg_have_dup_base values(-4, -4, -4, 'd')"; + Assertions.assertEquals(8, getOutputFragment(sql).getOutputExprs().size()); + } + @Test public void testAnalysisException() { String sql = "explain insert into t1(v1, v2) select k2 * 2, v1 + 1, v2 + 4 from src"; @@ -119,8 +129,8 @@ public class ExplainInsertCommandTest extends TestWithFeService { public void testWithMV() throws Exception { String sql = "explain insert into agg_have_dup_base select -4, -4, -4, 'd'"; Assertions.assertEquals(8, getOutputFragment(sql).getOutputExprs().size()); - String sql1 = "explain insert into agg_have_dup_base select -4, k2, -4, 'd' from agg_have_dup_base"; - Assertions.assertEquals(8, getOutputFragment(sql1).getOutputExprs().size()); + sql = "explain insert into agg_have_dup_base select -4, k2, -4, 'd' from agg_have_dup_base"; + Assertions.assertEquals(8, getOutputFragment(sql).getOutputExprs().size()); } private PlanFragment getOutputFragment(String sql) throws Exception { diff --git a/regression-test/data/nereids_p0/insert_into_table/insert_values.out b/regression-test/data/nereids_p0/insert_into_table/insert_values.out new file mode 100644 index 0000000000..b0ce0bb296 --- /dev/null +++ b/regression-test/data/nereids_p0/insert_into_table/insert_values.out @@ -0,0 +1,51 @@ +-- 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 + +-- !mv -- +-4 -4 -4 d + diff --git a/regression-test/suites/nereids_p0/insert_into_table/complex_insert.groovy b/regression-test/suites/nereids_p0/insert_into_table/complex_insert.groovy index cf6168c71a..7153fedcb5 100644 --- a/regression-test/suites/nereids_p0/insert_into_table/complex_insert.groovy +++ b/regression-test/suites/nereids_p0/insert_into_table/complex_insert.groovy @@ -208,5 +208,4 @@ suite('complex_insert') { sql 'insert into agg_have_dup_base select -4, -4, -4, \'d\'' sql 'sync' qt_mv 'select * from agg_have_dup_base' - } \ No newline at end of file diff --git a/regression-test/suites/nereids_p0/insert_into_table/insert_values.groovy b/regression-test/suites/nereids_p0/insert_into_table/insert_values.groovy new file mode 100644 index 0000000000..a3bc355d49 --- /dev/null +++ b/regression-test/suites/nereids_p0/insert_into_table/insert_values.groovy @@ -0,0 +1,118 @@ +// 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_into_values') { + 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 'use nereids_insert_into_table_test' + + def t1 = 'value_t1' + def t2 = 'value_t2' + def t3 = '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 ${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 ${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 ${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" + + sql "drop table if exists agg_have_dup_base_value" + + sql """ + create table agg_have_dup_base_value ( + k1 int null, + k2 int not null, + k3 bigint null, + k4 varchar(100) null + ) + duplicate key (k1, k2, k3) + distributed by hash(k1) buckets 3 + properties("replication_num" = "1"); + """ + + createMV("create materialized view k12s3m as select k1, sum(k2), max(k2) from agg_have_dup_base_value group by k1;") + + sql "insert into agg_have_dup_base_value values (-4, -4, -4, 'd')" + sql "sync" + qt_mv "select * from agg_have_dup_base_value" +} \ No newline at end of file --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org