You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@doris.apache.org by GitBox <gi...@apache.org> on 2022/04/28 14:16:17 UTC

[GitHub] [incubator-doris] morrySnow commented on a diff in pull request #9303: [feature](planner) Support MetaScan for global dict

morrySnow commented on code in PR #9303:
URL: https://github.com/apache/incubator-doris/pull/9303#discussion_r860916127


##########
fe/fe-core/src/main/java/org/apache/doris/catalog/FunctionSet.java:
##########
@@ -1310,6 +1310,8 @@ public void addBuiltinBothScalaAndVectorized(Function fn) {
 
     public static final String COUNT = "count";
     public static final String WINDOW_FUNNEL = "window_funnel";
+
+    public static final String DICT = "dict";

Review Comment:
   maybe `global_dict` is better?



##########
fe/fe-core/src/main/java/org/apache/doris/catalog/FunctionSet.java:
##########
@@ -1373,6 +1375,11 @@ private void initAggregateBuiltins() {
                 "",
                 true, false, true, true));
 
+        addBuiltin(AggregateFunction.createBuiltin(FunctionSet.DICT,
+                Lists.newArrayList(Type.VARCHAR), Type.VARCHAR, Type.VARCHAR,
+                true, false, false, false

Review Comment:
   the last arg should be true ?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.
+
+package org.apache.doris.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();

Review Comment:
   final ?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.
+
+package org.apache.doris.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {
+        List<TScanRangeLocations> resultList = Lists.newArrayList();
+        Collection<Partition> partitionCollection = olapTable.getPartitions();
+        for (Partition partition : partitionCollection) {
+            long visibleVersion = partition.getVisibleVersion();
+            String visibleVersionStr = String.valueOf(visibleVersion);
+            MaterializedIndex index = partition.getBaseIndex();
+            List<Tablet> tablets = index.getTablets();
+            for (Tablet tablet : tablets) {
+                long tabletId = tablet.getId();
+                TScanRangeLocations scanRangeLocations = new TScanRangeLocations();
+                TPaloScanRange paloRange = new TPaloScanRange();
+                paloRange.setDbName("");
+                paloRange.setSchemaHash("");
+                paloRange.setVersion(visibleVersionStr);
+                paloRange.setVersionHash("");
+                paloRange.setTabletId(tabletId);
+
+                // random shuffle List && only collect one copy
+                List<Replica> replicas = tablet.getQueryableReplicas(visibleVersion);
+                if (replicas.isEmpty()) {
+                    LOG.error("no queryable replica found in tablet {}. visible version {}",
+                        tabletId, visibleVersion);
+                    if (LOG.isDebugEnabled()) {
+                        for (Replica replica : tablet.getReplicas()) {
+                            LOG.debug("tablet {}, replica: {}", tabletId, replica.toString());
+                        }
+                    }
+                    throw new RuntimeException("Failed to get scan range, no queryable replica found in tablet: " + tabletId);
+                }
+
+                Collections.shuffle(replicas);
+                boolean tabletIsNull = true;
+                List<String> errs = Lists.newArrayList();
+                for (Replica replica : replicas) {
+                    Backend backend = Catalog.getCurrentSystemInfo().getBackend(replica.getBackendId());
+                    if (backend == null || !backend.isAlive()) {
+                        LOG.debug("backend {} not exists or is not alive for replica {}",
+                            replica.getBackendId(), replica.getId());
+                        errs.add(replica.getId() + "'s backend " + replica.getBackendId() + " does not exist or not alive");
+                        continue;
+                    }
+                    String ip = backend.getHost();
+                    int port = backend.getBePort();
+                    TScanRangeLocation scanRangeLocation = new TScanRangeLocation(new TNetworkAddress(ip, port));
+                    scanRangeLocation.setBackendId(replica.getBackendId());
+                    scanRangeLocations.addToLocations(scanRangeLocation);
+                    paloRange.addToHosts(new TNetworkAddress(ip, port));
+                    tabletIsNull = false;
+                }
+                if (tabletIsNull) {
+                    throw new RuntimeException(tabletId + " have no queryable replicas. err: " + Joiner.on(", ").join(errs));
+                }
+                TScanRange scanRange = new TScanRange();
+                scanRange.setPaloScanRange(paloRange);
+                scanRangeLocations.setScanRange(scanRange);
+
+                resultList.add(scanRangeLocations);
+            }
+        }
+        return resultList;
+    }
+
+    public void pullDictSlots(AggregateInfo aggInfo) {
+        Preconditions.checkState(aggInfo.getGroupingExprs().isEmpty());
+        List<FunctionCallExpr> funcExprList = aggInfo.getAggregateExprs();
+        for (FunctionCallExpr funcExpr : funcExprList) {
+            FunctionParams funcParams = funcExpr.getFnParams();
+            for (Expr expr : funcParams.exprs()) {
+                if (expr instanceof SlotRef && expr.isAnalyzed()) {

Review Comment:
   when thie function called, expr should be analyzed, so we need to check `expr.isAnalyzed` ?



##########
fe/fe-core/src/main/java/org/apache/doris/analysis/AggregateInfo.java:
##########
@@ -341,6 +341,12 @@ public AggregateInfo getMergeAggInfo() {
 
     public boolean isMerge() { return aggPhase_.isMerge(); }
     public boolean isDistinctAgg() { return secondPhaseDistinctAggInfo_ != null; }
+
+    public boolean isMetaScan() {
+        return aggregateExprs_.size() > 0 &&
+            aggregateExprs_.get(0).getFn().getFunctionName().getFunction().equals(FunctionSet.DICT);

Review Comment:
   what will happen if we use meta scan agg with normal agg?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1159,6 +1159,10 @@ private PlanNode createAggregationPlan(SelectStmt selectStmt, Analyzer analyzer,
                     aggInfo.getSecondPhaseDistinctAggInfo());
             newRoot.init(analyzer);
             Preconditions.checkState(newRoot.hasValidStats());
+        } else if (aggInfo.isMetaScan()) {

Review Comment:
   we need to throw exception here



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.
+
+package org.apache.doris.planner;
+

Review Comment:
   import order should be:
   
   doris
   
   third party
   
   java



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.
+
+package org.apache.doris.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {

Review Comment:
   add some comment to explain why are parameters not used



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.
+
+package org.apache.doris.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";

Review Comment:
   could MetaScanNode do meta scan on non-olap table ?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/MetaScanNode.java:
##########
@@ -0,0 +1,188 @@
+// 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.
+
+package org.apache.doris.planner;
+
+import com.google.common.base.Joiner;
+import com.google.common.base.Preconditions;
+import com.google.common.collect.Lists;
+import org.apache.doris.analysis.AggregateInfo;
+import org.apache.doris.analysis.Expr;
+import org.apache.doris.analysis.FunctionCallExpr;
+import org.apache.doris.analysis.FunctionParams;
+import org.apache.doris.analysis.SlotDescriptor;
+import org.apache.doris.analysis.SlotRef;
+import org.apache.doris.analysis.TupleDescriptor;
+import org.apache.doris.analysis.TupleId;
+import org.apache.doris.catalog.Catalog;
+import org.apache.doris.catalog.MaterializedIndex;
+import org.apache.doris.catalog.OlapTable;
+import org.apache.doris.catalog.Partition;
+import org.apache.doris.catalog.Replica;
+import org.apache.doris.catalog.Table;
+import org.apache.doris.catalog.Tablet;
+import org.apache.doris.system.Backend;
+import org.apache.doris.thrift.TExplainLevel;
+import org.apache.doris.thrift.TMetaScanNode;
+import org.apache.doris.thrift.TNetworkAddress;
+import org.apache.doris.thrift.TPaloScanRange;
+import org.apache.doris.thrift.TPlanNode;
+import org.apache.doris.thrift.TPlanNodeType;
+import org.apache.doris.thrift.TScanRange;
+import org.apache.doris.thrift.TScanRangeLocation;
+import org.apache.doris.thrift.TScanRangeLocations;
+import org.apache.logging.log4j.LogManager;
+import org.apache.logging.log4j.Logger;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.StringJoiner;
+
+
+public class MetaScanNode extends ScanNode {
+
+    private static final Logger LOG = LogManager.getLogger(MetaScanNode.class);
+
+    private static final String NODE_NAME = "META_SCAN";
+
+    private Map<Integer, Integer> slotIdToDictId = new HashMap<>();
+
+    private final OlapTable olapTable;
+
+    public MetaScanNode(PlanNodeId id,
+                        TupleDescriptor tupleDesc) {
+        super(id, tupleDesc, NODE_NAME);
+        this.olapTable = (OlapTable) tupleDesc.getTable();
+    }
+
+    @Override
+    public List<TScanRangeLocations> getScanRangeLocations(long maxScanRangeLength) {
+        List<TScanRangeLocations> resultList = Lists.newArrayList();
+        Collection<Partition> partitionCollection = olapTable.getPartitions();
+        for (Partition partition : partitionCollection) {
+            long visibleVersion = partition.getVisibleVersion();
+            String visibleVersionStr = String.valueOf(visibleVersion);
+            MaterializedIndex index = partition.getBaseIndex();
+            List<Tablet> tablets = index.getTablets();
+            for (Tablet tablet : tablets) {
+                long tabletId = tablet.getId();
+                TScanRangeLocations scanRangeLocations = new TScanRangeLocations();
+                TPaloScanRange paloRange = new TPaloScanRange();
+                paloRange.setDbName("");
+                paloRange.setSchemaHash("");
+                paloRange.setVersion(visibleVersionStr);
+                paloRange.setVersionHash("");
+                paloRange.setTabletId(tabletId);
+
+                // random shuffle List && only collect one copy
+                List<Replica> replicas = tablet.getQueryableReplicas(visibleVersion);
+                if (replicas.isEmpty()) {
+                    LOG.error("no queryable replica found in tablet {}. visible version {}",
+                        tabletId, visibleVersion);
+                    if (LOG.isDebugEnabled()) {
+                        for (Replica replica : tablet.getReplicas()) {
+                            LOG.debug("tablet {}, replica: {}", tabletId, replica.toString());
+                        }
+                    }
+                    throw new RuntimeException("Failed to get scan range, no queryable replica found in tablet: " + tabletId);
+                }
+
+                Collections.shuffle(replicas);
+                boolean tabletIsNull = true;
+                List<String> errs = Lists.newArrayList();
+                for (Replica replica : replicas) {
+                    Backend backend = Catalog.getCurrentSystemInfo().getBackend(replica.getBackendId());
+                    if (backend == null || !backend.isAlive()) {
+                        LOG.debug("backend {} not exists or is not alive for replica {}",
+                            replica.getBackendId(), replica.getId());
+                        errs.add(replica.getId() + "'s backend " + replica.getBackendId() + " does not exist or not alive");
+                        continue;
+                    }
+                    String ip = backend.getHost();
+                    int port = backend.getBePort();
+                    TScanRangeLocation scanRangeLocation = new TScanRangeLocation(new TNetworkAddress(ip, port));
+                    scanRangeLocation.setBackendId(replica.getBackendId());
+                    scanRangeLocations.addToLocations(scanRangeLocation);
+                    paloRange.addToHosts(new TNetworkAddress(ip, port));
+                    tabletIsNull = false;
+                }
+                if (tabletIsNull) {
+                    throw new RuntimeException(tabletId + " have no queryable replicas. err: " + Joiner.on(", ").join(errs));
+                }
+                TScanRange scanRange = new TScanRange();
+                scanRange.setPaloScanRange(paloRange);
+                scanRangeLocations.setScanRange(scanRange);
+
+                resultList.add(scanRangeLocations);
+            }
+        }
+        return resultList;
+    }
+
+    public void pullDictSlots(AggregateInfo aggInfo) {
+        Preconditions.checkState(aggInfo.getGroupingExprs().isEmpty());

Review Comment:
   do we verify it and throw exception before this check?



##########
fe/fe-core/src/main/java/org/apache/doris/analysis/TableRef.java:
##########
@@ -422,9 +424,11 @@ protected void analyzeHints() throws AnalysisException {
         }
         // Currently only 'PREAGGOPEN' is supported

Review Comment:
   should we modify this comment?



##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1159,6 +1159,10 @@ private PlanNode createAggregationPlan(SelectStmt selectStmt, Analyzer analyzer,
                     aggInfo.getSecondPhaseDistinctAggInfo());
             newRoot.init(analyzer);
             Preconditions.checkState(newRoot.hasValidStats());
+        } else if (aggInfo.isMetaScan()) {
+            Preconditions.checkState(root instanceof MetaScanNode);
+            MetaScanNode metaScanNode = (MetaScanNode) root;
+            metaScanNode.pullDictSlots(aggInfo);

Review Comment:
   we need to check agg function name and then do `pullDictSlots`



##########
fe/fe-core/src/main/java/org/apache/doris/planner/SingleNodePlanner.java:
##########
@@ -1683,6 +1687,11 @@ private PlanNode createScanNode(Analyzer analyzer, TableRef tblRef, SelectStmt s
 
         switch (tblRef.getTable().getType()) {
             case OLAP:
+                if (tblRef.isMetaScan()) {
+                    MetaScanNode metaScanNode = new MetaScanNode(ctx_.getNextNodeId(), tblRef.getDesc());

Review Comment:
   if we only could do meta scan on olap table, should we named this node to `OlapMeataScanNode` ?



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@doris.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


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