This is an automated email from the ASF dual-hosted git repository. morrysnow 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 73621bdb18 [enhance](Nereids) process DELETE_SIGN_COLUMN of OlapTable(#16030) 73621bdb18 is described below commit 73621bdb18c3705dc9ef2ab0ca9ae66fb45fd460 Author: 谢健 <jianx...@gmail.com> AuthorDate: Fri Jan 20 11:27:35 2023 +0800 [enhance](Nereids) process DELETE_SIGN_COLUMN of OlapTable(#16030) 1. add DELETE_SIGN_COLUMN in non-visible-columns in LogicalOlapScan 2. when the table has a delete sign, add a filter `delete_sign_coumn = 0` 3. use output slots and non-visible slots to bind slot --- .../doris/nereids/rules/analysis/BindRelation.java | 46 ++++++++++++++++++---- .../nereids/rules/analysis/BindSlotReference.java | 12 ++++-- .../doris/nereids/rules/analysis/CheckPolicy.java | 3 ++ .../nereids/trees/expressions/SlotReference.java | 4 ++ .../trees/plans/logical/LogicalOlapScan.java | 17 ++++++++ .../non_user_visiable_output.groovy | 42 ++++++++++++++++++++ 6 files changed, 112 insertions(+), 12 deletions(-) diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindRelation.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindRelation.java index 2f4269d055..6093817a66 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindRelation.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindRelation.java @@ -17,6 +17,7 @@ package org.apache.doris.nereids.rules.analysis; +import org.apache.doris.catalog.Column; import org.apache.doris.catalog.DatabaseIf; import org.apache.doris.catalog.Env; import org.apache.doris.catalog.OlapTable; @@ -25,6 +26,7 @@ import org.apache.doris.catalog.Table; import org.apache.doris.catalog.TableIf; import org.apache.doris.catalog.View; import org.apache.doris.catalog.external.HMSExternalTable; +import org.apache.doris.common.util.Util; import org.apache.doris.datasource.CatalogIf; import org.apache.doris.nereids.CascadesContext; import org.apache.doris.nereids.analyzer.UnboundRelation; @@ -33,8 +35,13 @@ import org.apache.doris.nereids.memo.Memo; 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.trees.expressions.EqualTo; +import org.apache.doris.nereids.trees.expressions.Expression; +import org.apache.doris.nereids.trees.expressions.Slot; +import org.apache.doris.nereids.trees.expressions.literal.TinyIntLiteral; import org.apache.doris.nereids.trees.plans.Plan; import org.apache.doris.nereids.trees.plans.logical.LogicalFileScan; +import org.apache.doris.nereids.trees.plans.logical.LogicalFilter; import org.apache.doris.nereids.trees.plans.logical.LogicalOlapScan; import org.apache.doris.nereids.trees.plans.logical.LogicalPlan; import org.apache.doris.nereids.trees.plans.logical.LogicalSchemaScan; @@ -42,8 +49,10 @@ import org.apache.doris.nereids.trees.plans.logical.LogicalSubQueryAlias; import org.apache.doris.nereids.trees.plans.logical.RelationUtil; import org.apache.doris.qe.ConnectContext; +import com.google.common.base.Preconditions; import com.google.common.collect.ImmutableList; import com.google.common.collect.Lists; +import com.google.common.collect.Sets; import org.apache.commons.collections.CollectionUtils; import java.util.Collections; @@ -148,19 +157,40 @@ public class BindRelation extends OneAnalysisRuleFactory { return getLogicalPlan(table, unboundRelation, tableQualifier, cascadesContext); } + private LogicalPlan makeOlapScan(TableIf table, UnboundRelation unboundRelation, List<String> tableQualifier) { + LogicalOlapScan scan; + List<Long> partIds = getPartitionIds(table, unboundRelation); + if (!CollectionUtils.isEmpty(partIds)) { + scan = new LogicalOlapScan(RelationUtil.newRelationId(), + (OlapTable) table, ImmutableList.of(tableQualifier.get(1)), partIds); + } else { + scan = new LogicalOlapScan(RelationUtil.newRelationId(), + (OlapTable) table, ImmutableList.of(tableQualifier.get(1))); + } + if (!Util.showHiddenColumns() && scan.getTable().hasDeleteSign() + && !ConnectContext.get().getSessionVariable() + .skipDeleteSign()) { + // table qualifier is catalog.db.table, we make db.table.column + Slot deleteSlot = null; + for (Slot slot : scan.getOutput()) { + if (slot.getName().equals(Column.DELETE_SIGN)) { + deleteSlot = slot; + break; + } + } + Preconditions.checkArgument(deleteSlot != null); + Expression conjunct = new EqualTo(new TinyIntLiteral((byte) 0), deleteSlot); + return new LogicalFilter(Sets.newHashSet(conjunct), scan); + } + return scan; + } + private LogicalPlan getLogicalPlan(TableIf table, UnboundRelation unboundRelation, List<String> tableQualifier, CascadesContext cascadesContext) { String dbName = tableQualifier.get(1); //[catalogName, dbName, tableName] switch (table.getType()) { case OLAP: - List<Long> partIds = getPartitionIds(table, unboundRelation); - if (!CollectionUtils.isEmpty(partIds)) { - return new LogicalOlapScan(RelationUtil.newRelationId(), - (OlapTable) table, ImmutableList.of(dbName), partIds); - } else { - return new LogicalOlapScan(RelationUtil.newRelationId(), - (OlapTable) table, ImmutableList.of(dbName)); - } + return makeOlapScan(table, unboundRelation, tableQualifier); case VIEW: Plan viewPlan = parseAndAnalyzeView(((View) table).getDdlSql(), cascadesContext); return new LogicalSubQueryAlias<>(tableQualifier, viewPlan); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotReference.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotReference.java index 97458c28d2..d1ed45f4ba 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotReference.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/BindSlotReference.java @@ -692,22 +692,26 @@ public class BindSlotReference implements AnalysisRuleFactory { @Override public Expression visitUnboundStar(UnboundStar unboundStar, PlannerContext context) { List<String> qualifier = unboundStar.getQualifier(); + List<Slot> slots = getScope().getSlots() + .stream() + .filter(slot -> !(slot instanceof SlotReference) || ((SlotReference) slot).isVisible()) + .collect(Collectors.toList()); switch (qualifier.size()) { case 0: // select * - return new BoundStar(getScope().getSlots()); + return new BoundStar(slots); case 1: // select table.* case 2: // select db.table.* - return bindQualifiedStar(qualifier); + return bindQualifiedStar(qualifier, slots); default: throw new AnalysisException("Not supported qualifier: " + StringUtils.join(qualifier, ".")); } } - private BoundStar bindQualifiedStar(List<String> qualifierStar) { + private BoundStar bindQualifiedStar(List<String> qualifierStar, List<Slot> boundSlots) { // FIXME: compatible with previous behavior: // https://github.com/apache/doris/pull/10415/files/3fe9cb0c3f805ab3a9678033b281b16ad93ec60a#r910239452 - List<Slot> slots = getScope().getSlots().stream().filter(boundSlot -> { + List<Slot> slots = boundSlots.stream().filter(boundSlot -> { switch (qualifierStar.size()) { // table.* case 1: diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckPolicy.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckPolicy.java index f062eff0b2..65df581bbf 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckPolicy.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/rules/analysis/CheckPolicy.java @@ -41,6 +41,9 @@ public class CheckPolicy implements AnalysisRuleFactory { RuleType.CHECK_ROW_POLICY.build( logicalCheckPolicy(logicalSubQueryAlias()).then(checkPolicy -> checkPolicy.child()) ), + RuleType.CHECK_ROW_POLICY.build( + logicalCheckPolicy(logicalFilter()).then(checkPolicy -> checkPolicy.child()) + ), RuleType.CHECK_ROW_POLICY.build( logicalCheckPolicy(logicalRelation()).thenApply(ctx -> { LogicalCheckPolicy<LogicalRelation> checkPolicy = ctx.root; diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java index acc536a3df..108bc481cd 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/expressions/SlotReference.java @@ -182,6 +182,10 @@ public class SlotReference extends Slot { return new SlotReference(exprId, name, dataType, nullable, qualifiers, column); } + public boolean isVisible() { + return column == null || column.isVisible(); + } + @Override public Slot withName(String name) { return new SlotReference(exprId, name, dataType, nullable, qualifier, column); diff --git a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java index b11b65d0cc..59b8ef5315 100644 --- a/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java +++ b/fe/fe-core/src/main/java/org/apache/doris/nereids/trees/plans/logical/LogicalOlapScan.java @@ -22,6 +22,7 @@ import org.apache.doris.catalog.Database; import org.apache.doris.catalog.Env; import org.apache.doris.catalog.OlapTable; import org.apache.doris.catalog.Table; +import org.apache.doris.common.util.Util; import org.apache.doris.nereids.exceptions.AnalysisException; import org.apache.doris.nereids.memo.GroupExpression; import org.apache.doris.nereids.properties.LogicalProperties; @@ -34,17 +35,20 @@ import org.apache.doris.nereids.trees.plans.algebra.CatalogRelation; import org.apache.doris.nereids.trees.plans.algebra.OlapScan; import org.apache.doris.nereids.trees.plans.visitor.PlanVisitor; import org.apache.doris.nereids.util.Utils; +import org.apache.doris.qe.ConnectContext; import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Preconditions; import com.google.common.collect.ImmutableList; +import java.util.ArrayList; import java.util.Collections; import java.util.List; import java.util.Objects; import java.util.Optional; import java.util.Set; import java.util.stream.Collectors; +import java.util.stream.Stream; /** * Logical OlapScan. @@ -262,6 +266,19 @@ public class LogicalOlapScan extends LogicalRelation implements CatalogRelation, : Optional.empty(); } + @Override + public List<Slot> computeOutput() { + List<Column> otherColumns = new ArrayList<>(); + if (!Util.showHiddenColumns() && getTable().hasDeleteSign() + && !ConnectContext.get().getSessionVariable() + .skipDeleteSign()) { + otherColumns.add(getTable().getDeleteSignColumn()); + } + return Stream.concat(table.getBaseSchema().stream(), otherColumns.stream()) + .map(col -> SlotReference.fromColumn(col, qualified())) + .collect(ImmutableList.toImmutableList()); + } + @Override public List<Slot> computeNonUserVisibleOutput() { Set<String> baseSchemaColNames = table.getBaseSchema().stream() diff --git a/regression-test/suites/nereids_syntax_p0/non_user_visiable_output.groovy b/regression-test/suites/nereids_syntax_p0/non_user_visiable_output.groovy new file mode 100644 index 0000000000..ced2318e14 --- /dev/null +++ b/regression-test/suites/nereids_syntax_p0/non_user_visiable_output.groovy @@ -0,0 +1,42 @@ +// 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("non_user_visiable_output") { + sql 'set enable_nereids_planner=true' + sql 'set enable_fallback_to_original_planner=false' + sql 'set enable_vectorized_engine=true' + sql """ + drop table if exists t_del; + """ + sql """ + create table t_del ( + id int not null + ) + UNIQUE KEY (id) + distributed by hash(id) + properties( + 'replication_num' = '1' + ); + """ + sql """insert into t_del values (1),(2),(3);""" + sql "delete from t_del where id = 2;" + test { + sql "select id from t_del order by id;" + result([[1],[3]]) + } + +} --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@doris.apache.org For additional commands, e-mail: commits-h...@doris.apache.org