Skip to content
Merged
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
13 changes: 13 additions & 0 deletions fe/fe-core/src/main/java/com/starrocks/qe/SessionVariable.java
Original file line number Diff line number Diff line change
Expand Up @@ -782,6 +782,8 @@ public static MaterializedViewRewriteMode parse(String str) {

public static final String ENABLE_PLAN_VALIDATION = "enable_plan_validation";

public static final String ENABLE_OPTIMIZER_RULE_DEBUG = "enable_optimizer_rule_debug";

public static final String ENABLE_STRICT_TYPE = "enable_strict_type";

public static final String PARTIAL_UPDATE_MODE = "partial_update_mode";
Expand Down Expand Up @@ -2463,6 +2465,9 @@ public long getConnectorSinkTargetMaxFileSize() {
@VarAttr(name = ENABLE_PLAN_VALIDATION, flag = VariableMgr.INVISIBLE)
private boolean enablePlanValidation = true;

@VarAttr(name = ENABLE_OPTIMIZER_RULE_DEBUG)
private boolean enableOptimizerRuleDebug = false;

@VarAttr(name = SCAN_OR_TO_UNION_LIMIT, flag = VariableMgr.INVISIBLE)
private int scanOrToUnionLimit = 4;

Expand Down Expand Up @@ -4731,6 +4736,14 @@ public void setEnablePlanValidation(boolean val) {
this.enablePlanValidation = val;
}

public boolean enableOptimizerRuleDebug() {
return this.enableOptimizerRuleDebug;
}

public void setEnableOptimizerRuleDebug(boolean enableOptimizerRuleDebug) {
this.enableOptimizerRuleDebug = enableOptimizerRuleDebug;
}

public boolean isCboPruneSubfield() {
return cboPruneSubfield;
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -131,7 +131,6 @@
import com.starrocks.sql.optimizer.task.TaskScheduler;
import com.starrocks.sql.optimizer.validate.MVRewriteValidator;
import com.starrocks.sql.optimizer.validate.OptExpressionValidator;
import com.starrocks.sql.optimizer.validate.PlanValidator;
import org.apache.commons.collections4.CollectionUtils;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
Expand Down Expand Up @@ -309,8 +308,9 @@ private OptExpression optimizeByCost(ConnectContext connectContext,
mvScan.stream().map(scan -> ((MaterializedView) scan.getTable()).getDbId()).forEach(currentSqlDbIds::add);

try (Timer ignored = Tracers.watchScope("PlanValidate")) {
rootTaskContext.getPlanValidator().enableAllCheckers();
// valid the final plan
PlanValidator.getInstance().validatePlan(finalPlan, rootTaskContext);
rootTaskContext.getPlanValidator().validatePlan(finalPlan, rootTaskContext);
// validate mv and log tracer if needed
MVRewriteValidator mvRewriteValidator = new MVRewriteValidator(allLogicalOlapScanOperators);
mvRewriteValidator.validateMV(connectContext, finalPlan, rootTaskContext);
Expand Down Expand Up @@ -589,6 +589,9 @@ private OptExpression logicalRuleRewrite(
rootTaskContext.setRequiredColumns(requiredColumns.clone());
scheduler.rewriteOnce(tree, rootTaskContext, RuleSet.PRUNE_COLUMNS_RULES);

// Enable input dependencies checker after column pruning for rule transformation validation
rootTaskContext.getPlanValidator().enableInputDependenciesChecker();

pruneTables(tree, rootTaskContext, requiredColumns);

scheduler.rewriteIterative(tree, rootTaskContext, new PruneEmptyWindowRule());
Expand Down Expand Up @@ -663,7 +666,6 @@ private OptExpression logicalRuleRewrite(

scheduler.rewriteDownTop(tree, rootTaskContext, OnPredicateMoveAroundRule.INSTANCE);
scheduler.rewriteIterative(tree, rootTaskContext, RuleSet.PUSH_DOWN_PREDICATE_RULES);

scheduler.rewriteIterative(tree, rootTaskContext, new PartitionColumnMinMaxRewriteRule());
scheduler.rewriteOnce(tree, rootTaskContext, RuleSet.PARTITION_PRUNE_RULES);
scheduler.rewriteIterative(tree, rootTaskContext, new RewriteMultiDistinctRule());
Expand Down Expand Up @@ -699,10 +701,8 @@ private OptExpression logicalRuleRewrite(
scheduler.rewriteOnce(tree, rootTaskContext, new PartitionColumnValueOnlyOnScanRule());
// before MergeProjectWithChildRule, after INLINE_CTE and MergeApplyWithTableFunction
scheduler.rewriteIterative(tree, rootTaskContext, RewriteUnnestBitmapRule.getInstance());

// After this rule, we shouldn't generate logical project operator
scheduler.rewriteIterative(tree, rootTaskContext, new MergeProjectWithChildRule());

scheduler.rewriteOnce(tree, rootTaskContext, new EliminateSortColumnWithEqualityPredicateRule());
scheduler.rewriteOnce(tree, rootTaskContext, new PushDownTopNBelowOuterJoinRule());
// intersect rewrite depend on statistics
Expand All @@ -720,7 +720,6 @@ private OptExpression logicalRuleRewrite(
scheduler.rewriteOnce(tree, rootTaskContext, JsonPathRewriteRule.createForOlapScan());
scheduler.rewriteIterative(tree, rootTaskContext, new RewriteMinMaxByMonotonicFunctionRule());
scheduler.rewriteOnce(tree, rootTaskContext, RewriteSimpleAggToHDFSScanRule.SCAN_NO_PROJECT);

// NOTE: This rule should be after MV Rewrite because MV Rewrite cannot handle
// select count(distinct c) from t group by a, b
// if this rule has applied before MV.
Expand Down Expand Up @@ -975,7 +974,6 @@ void memoOptimize(ConnectContext connectContext, Memo memo, TaskContext rootTask
private OptExpression physicalRuleRewrite(ConnectContext connectContext, TaskContext rootTaskContext,
OptExpression result) {
Preconditions.checkState(result.getOp().isPhysical());

int planCount = result.getPlanCount();

// Since there may be many different plans in the logic phase, it's possible
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -137,7 +137,7 @@ public List<ColumnOutputInfo> getCommonColInfo() {
}

public List<ColumnRefOperator> getOutputColRefs() {
return chooseOutputMap().values().stream().map(e -> e.getColumnRef()).collect(Collectors.toList());
return chooseOutputMap().values().stream().map(ColumnOutputInfo::getColumnRef).collect(Collectors.toList());
}

public Map<ColumnRefOperator, ScalarOperator> getColumnRefMap() {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -47,7 +47,7 @@ public ColumnRefSet(Collection<ColumnRefOperator> refs) {

public static ColumnRefSet createByIds(Collection<Integer> colIds) {
ColumnRefSet columnRefSet = new ColumnRefSet();
colIds.stream().forEach(columnRefSet::union);
colIds.forEach(columnRefSet::union);
return columnRefSet;
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -247,7 +247,7 @@ private OptExpression buildSelectConsume(ColumnRefFactory factory,
/*
* select a, b, c, d, null, sum(x) x from t group by rollup(a, b, c, d)
*/
private OptExpression buildSubRepeatConsume(ColumnRefFactory factory,
public OptExpression buildSubRepeatConsume(ColumnRefFactory factory,
Map<ColumnRefOperator, ColumnRefOperator> outputs,
LogicalAggregationOperator aggregate, LogicalRepeatOperator repeat,
int cteId) {
Expand Down Expand Up @@ -357,4 +357,4 @@ private OptExpression buildSubRepeatConsume(ColumnRefFactory factory,
return OptExpression.create(projectOperator,
OptExpression.create(newAggregate, OptExpression.create(newRepeat, OptExpression.create(consume))));
}
}
}
Original file line number Diff line number Diff line change
Expand Up @@ -89,6 +89,7 @@ public void execute() {
}
List<OptExpression> targetExpressions;
OptimizerTraceUtil.logApplyRuleBefore(context.getOptimizerContext(), rule, extractExpr);

try (Timer ignore = Tracers.watchScope(Tracers.Module.OPTIMIZER, rule.getClass().getSimpleName())) {
targetExpressions = rule.transform(extractExpr, context.getOptimizerContext());
} catch (StarRocksPlannerException e) {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -23,6 +23,7 @@
import com.starrocks.sql.optimizer.operator.OperatorType;
import com.starrocks.sql.optimizer.operator.pattern.Pattern;
import com.starrocks.sql.optimizer.rule.Rule;
import com.starrocks.sql.optimizer.validate.PlanValidator;

import java.util.List;

Expand Down Expand Up @@ -62,8 +63,22 @@ public void execute() {
.getOptimizerOptions().isRuleDisable(rule.type()))) {
return;
}

// Save whole tree state before rule group execution for validation
OptExpression wholePlanBefore = null;
if (context.getOptimizerContext().getSessionVariable().enableOptimizerRuleDebug()) {
wholePlanBefore = planTree.getInputs().get(0);
}

// first node must be RewriteAnchorNode
rewrite(planTree, 0, planTree.getInputs().get(0));

// Validate after the entire rule group execution
if (wholePlanBefore != null && change > 0) {
OptExpression wholePlanAfter = planTree.getInputs().get(0);
PlanValidator.validateAfterRule(wholePlanBefore, wholePlanAfter, rules.get(0), context);
}

// pushdownNotNullPredicates should task-bind, reset it before another RewriteTreeTask
// TODO: refactor TaskContext to make it local to support this requirement better?
context.getOptimizerContext().clearNotNullPredicates();
Expand Down Expand Up @@ -97,6 +112,7 @@ protected OptExpression applyRules(OptExpression parent, int childIndex, OptExpr
}

OptimizerTraceUtil.logApplyRuleBefore(context.getOptimizerContext(), rule, root);

List<OptExpression> result;
try (Timer ignore = Tracers.watchScope(Tracers.Module.OPTIMIZER, rule.toString())) {
result = rule.transform(root, context.getOptimizerContext());
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -18,13 +18,15 @@
import com.starrocks.sql.optimizer.OptimizerContext;
import com.starrocks.sql.optimizer.base.ColumnRefSet;
import com.starrocks.sql.optimizer.base.PhysicalPropertySet;
import com.starrocks.sql.optimizer.validate.PlanValidator;

// The context for optimizer task
public class TaskContext {
private final OptimizerContext optimizerContext;
private final PhysicalPropertySet requiredProperty;
private ColumnRefSet requiredColumns;
private double upperBoundCost;
private final PlanValidator planValidator;

public TaskContext(OptimizerContext context,
PhysicalPropertySet physicalPropertySet,
Expand All @@ -34,6 +36,11 @@ public TaskContext(OptimizerContext context,
this.requiredProperty = physicalPropertySet;
this.requiredColumns = requiredColumns;
this.upperBoundCost = cost;
this.planValidator = new PlanValidator();
}

public PlanValidator getPlanValidator() {
return planValidator;
}

public OptimizerContext getOptimizerContext() {
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -43,7 +43,6 @@ public static CTEUniqueChecker getInstance() {
return INSTANCE;
}


@Override
public void validate(OptExpression physicalPlan, TaskContext taskContext) {
CTEUniqueChecker.Visitor visitor = new CTEUniqueChecker.Visitor();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -202,15 +202,15 @@ private void checkInputCols(ColumnRefSet inputCols, ColumnRefSet usedCols, OptEx
ColumnRefSet missedCols = usedCols.clone();
missedCols.except(inputCols);
if (!missedCols.isEmpty()) {
String message = String.format("Invalid plan:%s%s%s The required cols %s cannot obtain from input cols %s.",
String message = String.format("Invalid plan:%s%s\n%s \nThe required cols %s cannot obtain from input cols %s.",
System.lineSeparator(), optExpr.debugString(), PREFIX, missedCols, inputCols);
throw new StarRocksPlannerException(message, ErrorType.INTERNAL_ERROR);
}
}

private void checkInputType(ColumnRefOperator inputCol, ColumnRefOperator outputCol, OptExpression optExpression) {
if (!outputCol.getType().isFullyCompatible(inputCol.getType())) {
String message = String.format("Invalid plan:%s%s%s Type of output col %s is not fully compatible with " +
String message = String.format("Invalid plan:%s%s\n%s \nType of output col %s is not fully compatible with " +
"type of input col %s.",
System.lineSeparator(), optExpression.debugString(), PREFIX, outputCol, inputCol);
throw new StarRocksPlannerException(message, ErrorType.INTERNAL_ERROR);
Expand All @@ -219,7 +219,7 @@ private void checkInputType(ColumnRefOperator inputCol, ColumnRefOperator output

private void checkChildNumberOfSet(int inputSize, int requiredSize, OptExpression optExpression) {
if (inputSize != requiredSize) {
String message = String.format("Invalid plan:%s%s%s. The required number of children is %d but found %d.",
String message = String.format("Invalid plan:%s%s\n%s. \nThe required number of children is %d but found %d.",
System.lineSeparator(), optExpression.debugString(), PREFIX, requiredSize, inputSize);
throw new StarRocksPlannerException(message, ErrorType.INTERNAL_ERROR);
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -21,6 +21,7 @@
import com.starrocks.sql.common.ErrorType;
import com.starrocks.sql.common.StarRocksPlannerException;
import com.starrocks.sql.optimizer.OptExpression;
import com.starrocks.sql.optimizer.rule.Rule;
import com.starrocks.sql.optimizer.task.TaskContext;
import org.apache.logging.log4j.LogManager;
import org.apache.logging.log4j.Logger;
Expand All @@ -31,22 +32,35 @@ public final class PlanValidator {

private static final Logger LOGGER = LogManager.getLogger(PlanValidator.class);

private static final PlanValidator INSTANCE = new PlanValidator();
private List<Checker> checkerList;
private boolean enableInputDependenciesChecker = false;

private final List<Checker> checkerList;
public PlanValidator() {
checkerList = ImmutableList.of(CTEUniqueChecker.getInstance());
}

/**
* Enable inputDependencies checker for rule transformation phase.
* This method should be called after column pruning to ensure InputDependenciesChecker
* can properly validate column dependencies in the optimized plan.
*/
public void enableInputDependenciesChecker() {
if (!enableInputDependenciesChecker) {
checkerList = ImmutableList.of(
CTEUniqueChecker.getInstance(),
InputDependenciesChecker.getInstance());
enableInputDependenciesChecker = true;
}
}

private PlanValidator() {
public void enableAllCheckers() {
checkerList = ImmutableList.of(
InputDependenciesChecker.getInstance(),
TypeChecker.getInstance(),
CTEUniqueChecker.getInstance(),
InputDependenciesChecker.getInstance(),
ColumnReuseChecker.getInstance());
}

public static PlanValidator getInstance() {
return INSTANCE;
}

public void validatePlan(OptExpression optExpression, TaskContext taskContext) {
boolean enablePlanValidation = ConnectContext.get().getSessionVariable().getEnablePlanValidation();
try {
Expand All @@ -58,7 +72,7 @@ public void validatePlan(OptExpression optExpression, TaskContext taskContext) {
} catch (IllegalArgumentException e) {
String message = e.getMessage();
if (!message.contains("Invalid plan")) {
message = "Invalid plan:\n" + optExpression.debugString() + message;
message = "Invalid plan:\n" + optExpression.debugString() + "\n" + message;
}
LOGGER.debug("Failed to validate plan.", e);
if (enablePlanValidation) {
Expand All @@ -77,6 +91,44 @@ public void validatePlan(OptExpression optExpression, TaskContext taskContext) {
}
}

/**
* Validates the query plan after applying an optimizer rule for debugging purposes.
* Provides detailed error information including rule name and before/after expressions.
*
* @param beforeExpression The query plan expression before rule transformation
* @param afterExpression The query plan expression after rule transformation
* @param rule The optimizer rule that was applied
* @param taskContext The task context containing session variables
* @throws IllegalStateException if validation fails, with detailed rule information
*/
public static void validateAfterRule(OptExpression beforeExpression, OptExpression afterExpression,
Rule rule, TaskContext taskContext) {
if (!taskContext.getOptimizerContext().getSessionVariable().enableOptimizerRuleDebug()) {
return;
}

try {
OptExpression clonedOpt = OptExpression.create(afterExpression.getOp(), afterExpression.getInputs());
clonedOpt.clearStatsAndInitOutputInfo();
taskContext.getPlanValidator().validatePlan(clonedOpt, taskContext);
} catch (Exception e) {
String errorMsg = String.format("Optimizer rule debug: Plan validation failed after applying rule [%s].\n" +
"Validation error: %s\n" +
"Hint: This error was caught by enable_optimizer_rule_debug=true\n" +
"\n=== Query Plan Before Rule Application ===\n%s\n" +
"\n=== Query Plan After Rule Application ===\n%s\n",
rule.type(),
e.getMessage(),
beforeExpression.debugString(),
afterExpression.debugString()
);
LOGGER.error(errorMsg, e);

throw new IllegalStateException(errorMsg, e);
}
}


public interface Checker {
void validate(OptExpression physicalPlan, TaskContext taskContext);
}
Expand Down
Loading
Loading