Skip to content

Commit

Permalink
Add explain support, perfect the return type of diff function
Browse files Browse the repository at this point in the history
  • Loading branch information
Beyyes authored Jun 15, 2024
1 parent 7ef70c7 commit bbb7846
Show file tree
Hide file tree
Showing 17 changed files with 259 additions and 631 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -100,6 +100,10 @@ public StatementMemorySource visitExplain(
new PlanGraphPrinter.GraphContext(
context.getQueryContext().getTypeProvider().getTemplatedInfo()));

return getStatementMemorySource(header, lines);
}

static StatementMemorySource getStatementMemorySource(DatasetHeader header, List<String> lines) {
TsBlockBuilder builder = new TsBlockBuilder(Collections.singletonList(TSDataType.TEXT));
lines.forEach(
line -> {
Expand Down
Original file line number Diff line number Diff line change
@@ -0,0 +1,41 @@
/*
* 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.iotdb.db.queryengine.plan.execution.memory;

import org.apache.iotdb.db.queryengine.common.MPPQueryContext;
import org.apache.iotdb.db.queryengine.plan.relational.analyzer.Analysis;

public class TableModelStatementMemorySourceContext {
private final MPPQueryContext queryContext;
private final Analysis analysis;

public TableModelStatementMemorySourceContext(MPPQueryContext queryContext, Analysis analysis) {
this.queryContext = queryContext;
this.analysis = analysis;
}

public MPPQueryContext getQueryContext() {
return queryContext;
}

public Analysis getAnalysis() {
return analysis;
}
}
Original file line number Diff line number Diff line change
@@ -0,0 +1,94 @@
/*
* 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.iotdb.db.queryengine.plan.execution.memory;

import org.apache.iotdb.commons.conf.IoTDBConstant;
import org.apache.iotdb.db.queryengine.common.header.ColumnHeader;
import org.apache.iotdb.db.queryengine.common.header.DatasetHeader;
import org.apache.iotdb.db.queryengine.execution.warnings.WarningCollector;
import org.apache.iotdb.db.queryengine.plan.analyze.ClusterPartitionFetcher;
import org.apache.iotdb.db.queryengine.plan.planner.LocalExecutionPlanner;
import org.apache.iotdb.db.queryengine.plan.planner.plan.LogicalQueryPlan;
import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanGraphPrinter;
import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanNode;
import org.apache.iotdb.db.queryengine.plan.relational.planner.distribute.ExchangeNodeGenerator;
import org.apache.iotdb.db.queryengine.plan.relational.sql.ast.AstVisitor;
import org.apache.iotdb.db.queryengine.plan.relational.sql.ast.Explain;
import org.apache.iotdb.db.queryengine.plan.relational.sql.ast.Node;

import org.apache.tsfile.enums.TSDataType;
import org.apache.tsfile.read.common.block.TsBlock;

import java.util.Collections;
import java.util.List;

import static org.apache.iotdb.db.queryengine.common.header.DatasetHeader.EMPTY_HEADER;
import static org.apache.iotdb.db.queryengine.plan.execution.memory.StatementMemorySourceVisitor.getStatementMemorySource;

public class TableModelStatementMemorySourceVisitor
extends AstVisitor<StatementMemorySource, TableModelStatementMemorySourceContext> {

@Override
public StatementMemorySource visitNode(
Node node, TableModelStatementMemorySourceContext context) {
DatasetHeader datasetHeader = context.getAnalysis().getRespDatasetHeader();
return new StatementMemorySource(
new TsBlock(0), datasetHeader == null ? EMPTY_HEADER : datasetHeader);
}

@Override
public StatementMemorySource visitExplain(
Explain node, TableModelStatementMemorySourceContext context) {
context.getAnalysis().setStatement(node.getStatement());
DatasetHeader header =
new DatasetHeader(
Collections.singletonList(
new ColumnHeader(IoTDBConstant.COLUMN_DISTRIBUTION_PLAN, TSDataType.TEXT)),
true);
LogicalQueryPlan logicalPlan =
new org.apache.iotdb.db.queryengine.plan.relational.planner.LogicalPlanner(
context.getQueryContext(),
LocalExecutionPlanner.getInstance().metadata,
context.getQueryContext().getSession(),
ClusterPartitionFetcher.getInstance(),
WarningCollector.NOOP)
.plan(context.getAnalysis());
if (context.getAnalysis().getDataPartition() == null
|| context.getAnalysis().getDataPartition().isEmpty()) {
return new StatementMemorySource(new TsBlock(0), header);
}

// TODO(beyyes) adapt this logic after optimize ExchangeNodeAdder
ExchangeNodeGenerator.PlanContext exchangeContext =
new ExchangeNodeGenerator.PlanContext(context.getQueryContext());
List<PlanNode> distributedPlanNodeResult =
new ExchangeNodeGenerator().visitPlan(logicalPlan.getRootNode(), exchangeContext);

List<String> lines =
distributedPlanNodeResult
.get(0)
.accept(
new PlanGraphPrinter(),
new PlanGraphPrinter.GraphContext(
context.getQueryContext().getTypeProvider().getTemplatedInfo()));

return getStatementMemorySource(header, lines);
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -66,6 +66,7 @@
import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.CrossSeriesAggregationDescriptor;
import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.DeviceViewIntoPathDescriptor;
import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.IntoPathDescriptor;
import org.apache.iotdb.db.queryengine.plan.relational.planner.node.TableScanNode;

import org.apache.commons.lang3.Validate;
import org.apache.tsfile.utils.Pair;
Expand Down Expand Up @@ -598,6 +599,59 @@ public List<String> visitColumnInject(ColumnInjectNode node, GraphContext contex
return render(node, boxValue, context);
}

// =============== Methods below are used for table model ================
@Override
public List<String> visitTableScan(TableScanNode node, GraphContext context) {
List<String> boxValue = new ArrayList<>();
boxValue.add(String.format("TableScan-%s", node.getPlanNodeId().getId()));
boxValue.add(String.format("QualifiedTableName: %s", node.getQualifiedTableName()));
boxValue.add(String.format("OutputSymbols: %s", node.getOutputSymbols()));
boxValue.add(String.format("DeviceEntriesSize: %s", node.getDeviceEntries().size()));
boxValue.add(String.format("ScanOrder: %s", node.getScanOrder()));
if (node.getPushDownPredicate() != null) {
boxValue.add(String.format("PushDownPredicate: %s", node.getPushDownPredicate()));
}
if (node.getPushDownOffset() > 0) {
boxValue.add(String.format("PushDownOffset: %s", node.getPushDownOffset()));
}
if (node.getPushDownLimit() > 0) {
boxValue.add(String.format("PushDownLimit: %s", node.getPushDownLimit()));
}
boxValue.add(String.format("RegionId: %s", node.getRegionReplicaSet().getRegionId().getId()));
return render(node, boxValue, context);
}

@Override
public List<String> visitFilter(
org.apache.iotdb.db.queryengine.plan.relational.planner.node.FilterNode node,
GraphContext context) {
List<String> boxValue = new ArrayList<>();
boxValue.add(String.format("Filter-%s", node.getPlanNodeId().getId()));
boxValue.add(String.format("Predicate: %s", node.getPredicate()));
return render(node, boxValue, context);
}

@Override
public List<String> visitProject(
org.apache.iotdb.db.queryengine.plan.relational.planner.node.ProjectNode node,
GraphContext context) {
List<String> boxValue = new ArrayList<>();
boxValue.add(String.format("Project-%s", node.getPlanNodeId().getId()));
boxValue.add(String.format("OutputSymbols: %s", node.getOutputSymbols()));
boxValue.add(String.format("Expressions: %s", node.getAssignments().getMap().values()));
return render(node, boxValue, context);
}

@Override
public List<String> visitOutput(
org.apache.iotdb.db.queryengine.plan.relational.planner.node.OutputNode node,
GraphContext context) {
List<String> boxValue = new ArrayList<>();
boxValue.add(String.format("Output-%s", node.getPlanNodeId().getId()));
boxValue.add(String.format("OutputSymbols: %s", node.getOutputSymbols()));
return render(node, boxValue, context);
}

private String printRegion(TRegionReplicaSet regionReplicaSet) {
return String.format(
"Partition: %s",
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -27,6 +27,8 @@
import org.apache.iotdb.db.queryengine.common.header.DatasetHeader;
import org.apache.iotdb.db.queryengine.plan.analyze.IAnalysis;
import org.apache.iotdb.db.queryengine.plan.execution.memory.StatementMemorySource;
import org.apache.iotdb.db.queryengine.plan.execution.memory.TableModelStatementMemorySourceContext;
import org.apache.iotdb.db.queryengine.plan.execution.memory.TableModelStatementMemorySourceVisitor;
import org.apache.iotdb.db.queryengine.plan.planner.plan.TimePredicate;
import org.apache.iotdb.db.queryengine.plan.relational.metadata.QualifiedObjectName;
import org.apache.iotdb.db.queryengine.plan.relational.metadata.TableSchema;
Expand Down Expand Up @@ -86,11 +88,10 @@
import static java.util.Collections.unmodifiableMap;
import static java.util.Collections.unmodifiableSet;
import static java.util.Objects.requireNonNull;
import static org.apache.iotdb.db.queryengine.common.header.DatasetHeader.EMPTY_HEADER;

public class Analysis implements IAnalysis {

@Nullable private final Statement root;
@Nullable private Statement root;

private final Map<NodeRef<Parameter>, Expression> parameters;

Expand Down Expand Up @@ -269,6 +270,10 @@ public Scope getRootScope() {
return getScope(root);
}

public void setStatement(Statement statement) {
this.root = statement;
}

public void setScope(Node node, Scope scope) {
scopes.put(NodeRef.of(node), scope);
}
Expand Down Expand Up @@ -614,11 +619,12 @@ private boolean hasDataSource() {

@Override
public TsBlock constructResultForMemorySource(MPPQueryContext context) {
StatementMemorySource source =
new StatementMemorySource(
new TsBlock(0), respDatasetHeader == null ? EMPTY_HEADER : respDatasetHeader);
setRespDatasetHeader(source.getDatasetHeader());
return source.getTsBlock();
requireNonNull(getStatement(), "root statement is analysis is null");
StatementMemorySource memorySource =
new TableModelStatementMemorySourceVisitor()
.process(getStatement(), new TableModelStatementMemorySourceContext(context, this));
setRespDatasetHeader(memorySource.getDatasetHeader());
return memorySource.getTsBlock();
}

@Override
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -359,7 +359,8 @@ protected Scope visitDelete(Delete node, Optional<Scope> scope) {

@Override
protected Scope visitExplain(Explain node, Optional<Scope> context) {
throw new SemanticException("Explain statement is not supported yet.");
analysis.setFinishQueryAfterAnalyze();
return visitQuery((Query) node.getStatement(), context);
}

@Override
Expand Down

This file was deleted.

Loading

0 comments on commit bbb7846

Please sign in to comment.