Skip to content

Commit

Permalink
This is an automated cherry-pick of #45538
Browse files Browse the repository at this point in the history
Signed-off-by: ti-chi-bot <ti-community-prow-bot@tidb.io>
  • Loading branch information
qw4990 authored and ti-chi-bot committed Jul 28, 2023
1 parent 230b5ff commit 8cb5e66
Show file tree
Hide file tree
Showing 8 changed files with 612 additions and 67 deletions.
35 changes: 35 additions & 0 deletions planner/core/casetest/rule/BUILD.bazel
Original file line number Diff line number Diff line change
@@ -0,0 +1,35 @@
load("@io_bazel_rules_go//go:def.bzl", "go_test")

go_test(
name = "rule_test",
timeout = "short",
srcs = [
"main_test.go",
"rule_derive_topn_from_window_test.go",
"rule_inject_extra_projection_test.go",
"rule_join_reorder_test.go",
"rule_result_reorder_test.go",
],
data = glob(["testdata/**"]),
flaky = True,
shard_count = 21,
deps = [
"//domain",
"//expression",
"//expression/aggregation",
"//parser/ast",
"//parser/model",
"//parser/mysql",
"//planner/core/internal",
"//sessionctx/variable",
"//testkit",
"//testkit/testdata",
"//testkit/testmain",
"//testkit/testsetup",
"//types",
"//util/mock",
"@com_github_pingcap_failpoint//:failpoint",
"@com_github_stretchr_testify//require",
"@org_uber_go_goleak//:goleak",
],
)
316 changes: 316 additions & 0 deletions planner/core/casetest/rule/rule_join_reorder_test.go

Large diffs are not rendered by default.

12 changes: 11 additions & 1 deletion planner/core/exhaust_physical_plans.go
Original file line number Diff line number Diff line change
Expand Up @@ -229,7 +229,8 @@ func (p *LogicalJoin) GetMergeJoin(prop *property.PhysicalProperty, schema *expr
}
// If TiDB_SMJ hint is existed, it should consider enforce merge join,
// because we can't trust lhsChildProperty completely.
if (p.preferJoinType & preferMergeJoin) > 0 {
if (p.preferJoinType&preferMergeJoin) > 0 ||
(p.preferJoinType&preferNoHashJoin) > 0 { // if hash join is not allowed, generate as many other types of join as possible to avoid 'cant-find-plan' error.
joins = append(joins, p.getEnforcedMergeJoin(prop, schema, statsInfo)...)
}

Expand Down Expand Up @@ -387,6 +388,7 @@ func (p *LogicalJoin) getHashJoins(prop *property.PhysicalProperty) (joins []Phy
forceLeftToBuild = false
forceRightToBuild = false
}

joins = make([]PhysicalPlan, 0, 2)
switch p.JoinType {
case SemiJoin, AntiSemiJoin, LeftOuterSemiJoin, AntiLeftOuterSemiJoin:
Expand Down Expand Up @@ -433,7 +435,15 @@ func (p *LogicalJoin) getHashJoins(prop *property.PhysicalProperty) (joins []Phy
}
}
}

forced = (p.preferJoinType&preferHashJoin > 0) || forceLeftToBuild || forceRightToBuild
noHashJoin := (p.preferJoinType & preferNoHashJoin) > 0
if !forced && noHashJoin {
return nil, false
} else if forced && noHashJoin {
p.ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack(
"Some HASH_JOIN and NO_HASH_JOIN hints conflict, NO_HASH_JOIN is ignored"))
}
return
}

Expand Down
18 changes: 17 additions & 1 deletion planner/core/logical_plan_builder.go
Original file line number Diff line number Diff line change
Expand Up @@ -95,6 +95,8 @@ const (
HintINLMJ = "inl_merge_join"
// TiDBHashJoin is hint enforce hash join.
TiDBHashJoin = "tidb_hj"
// HintNoHashJoin is the hint to enforce the query not to use hash join.
HintNoHashJoin = "no_hash_join"
// HintHJ is hint enforce hash join.
HintHJ = "hash_join"
// HintHashJoinBuild is hint enforce hash join's build side
Expand Down Expand Up @@ -623,6 +625,14 @@ func (p *LogicalJoin) setPreferredJoinTypeAndOrder(hintInfo *tableHintInfo) {
p.preferJoinType |= preferHashJoin
p.rightPreferJoinType |= preferHashJoin
}
if hintInfo.ifPreferNoHashJoin(lhsAlias) {
p.preferJoinType |= preferNoHashJoin
p.leftPreferJoinType |= preferNoHashJoin
}
if hintInfo.ifPreferNoHashJoin(rhsAlias) {
p.preferJoinType |= preferNoHashJoin
p.rightPreferJoinType |= preferNoHashJoin
}
if hintInfo.ifPreferINLJ(lhsAlias) {
p.preferJoinType |= preferLeftAsINLJInner
p.leftPreferJoinType |= preferINLJ
Expand Down Expand Up @@ -3688,6 +3698,7 @@ func (b *PlanBuilder) pushTableHints(hints []*ast.TableOptimizerHint, currentLev
hints = b.hintProcessor.GetCurrentStmtHints(hints, currentLevel)
var (
sortMergeTables, inljTables, inlhjTables, inlmjTables, hashJoinTables, bcTables []hintTableInfo
noHashJoinTables []hintTableInfo
shuffleJoinTables []hintTableInfo
indexHintList, indexMergeHintList []indexHintInfo
tiflashTables, tikvTables []hintTableInfo
Expand All @@ -3702,7 +3713,7 @@ func (b *PlanBuilder) pushTableHints(hints []*ast.TableOptimizerHint, currentLev
for _, hint := range hints {
// Set warning for the hint that requires the table name.
switch hint.HintName.L {
case TiDBMergeJoin, HintSMJ, TiDBIndexNestedLoopJoin, HintINLJ, HintINLHJ, HintINLMJ,
case TiDBMergeJoin, HintSMJ, TiDBIndexNestedLoopJoin, HintINLJ, HintINLHJ, HintINLMJ, HintNoHashJoin,
TiDBHashJoin, HintHJ, HintUseIndex, HintIgnoreIndex, HintForceIndex, HintOrderIndex, HintNoOrderIndex, HintIndexMerge, HintLeading:
if len(hint.Tables) == 0 {
b.pushHintWithoutTableWarning(hint)
Expand All @@ -3725,6 +3736,8 @@ func (b *PlanBuilder) pushTableHints(hints []*ast.TableOptimizerHint, currentLev
inlmjTables = append(inlmjTables, tableNames2HintTableInfo(b.ctx, hint.HintName.L, hint.Tables, b.hintProcessor, currentLevel)...)
case TiDBHashJoin, HintHJ:
hashJoinTables = append(hashJoinTables, tableNames2HintTableInfo(b.ctx, hint.HintName.L, hint.Tables, b.hintProcessor, currentLevel)...)
case HintNoHashJoin:
noHashJoinTables = append(noHashJoinTables, tableNames2HintTableInfo(b.ctx, hint.HintName.L, hint.Tables, b.hintProcessor, currentLevel)...)
case HintMPP1PhaseAgg:
aggHints.preferAggType |= preferMPP1PhaseAgg
case HintMPP2PhaseAgg:
Expand Down Expand Up @@ -3835,6 +3848,7 @@ func (b *PlanBuilder) pushTableHints(hints []*ast.TableOptimizerHint, currentLev
shuffleJoinTables: shuffleJoinTables,
indexNestedLoopJoinTables: indexNestedLoopJoinTables{inljTables, inlhjTables, inlmjTables},
hashJoinTables: hashJoinTables,
noHashJoinTables: noHashJoinTables,
indexHintList: indexHintList,
tiflashTables: tiflashTables,
tikvTables: tikvTables,
Expand Down Expand Up @@ -7049,6 +7063,8 @@ func getInnerFromParenthesesAndUnaryPlus(expr ast.ExprNode) ast.ExprNode {
// containDifferentJoinTypes checks whether `preferJoinType` contains different
// join types.
func containDifferentJoinTypes(preferJoinType uint) bool {
preferJoinType &= ^preferNoHashJoin

inlMask := preferRightAsINLJInner ^ preferLeftAsINLJInner
inlhjMask := preferRightAsINLHJInner ^ preferLeftAsINLHJInner
inlmjMask := preferRightAsINLMJInner ^ preferLeftAsINLMJInner
Expand Down
1 change: 1 addition & 0 deletions planner/core/logical_plans.go
Original file line number Diff line number Diff line change
Expand Up @@ -117,6 +117,7 @@ const (
preferHJBuild
preferHJProbe
preferHashJoin
preferNoHashJoin
preferMergeJoin
preferBCJoin
preferShuffleJoin
Expand Down
5 changes: 5 additions & 0 deletions planner/core/planbuilder.go
Original file line number Diff line number Diff line change
Expand Up @@ -93,6 +93,7 @@ type tableHintInfo struct {
broadcastJoinTables []hintTableInfo
shuffleJoinTables []hintTableInfo
hashJoinTables []hintTableInfo
noHashJoinTables []hintTableInfo
indexHintList []indexHintInfo
tiflashTables []hintTableInfo
tikvTables []hintTableInfo
Expand Down Expand Up @@ -237,6 +238,10 @@ func (info *tableHintInfo) ifPreferHashJoin(tableNames ...*hintTableInfo) bool {
return info.matchTableName(tableNames, info.hashJoinTables)
}

func (info *tableHintInfo) ifPreferNoHashJoin(tableNames ...*hintTableInfo) bool {
return info.matchTableName(tableNames, info.noHashJoinTables)
}

func (info *tableHintInfo) ifPreferHJBuild(tableNames ...*hintTableInfo) bool {
return info.matchTableName(tableNames, info.hjBuildTables)
}
Expand Down
Loading

0 comments on commit 8cb5e66

Please sign in to comment.