Skip to content

Commit

Permalink
planner: support no_hash_join hint on optimizer (#45538)
Browse files Browse the repository at this point in the history
ref #45520
  • Loading branch information
qw4990 committed Jul 24, 2023
1 parent 7556233 commit 7c2dbbe
Show file tree
Hide file tree
Showing 8 changed files with 264 additions and 69 deletions.
2 changes: 1 addition & 1 deletion planner/core/casetest/rule/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -12,7 +12,7 @@ go_test(
],
data = glob(["testdata/**"]),
flaky = True,
shard_count = 20,
shard_count = 21,
deps = [
"//domain",
"//expression",
Expand Down
11 changes: 11 additions & 0 deletions planner/core/casetest/rule/rule_join_reorder_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -61,6 +61,17 @@ func TestStraightJoinHint(t *testing.T) {
runJoinReorderTestData(t, tk, "TestStraightJoinHint")
}

func TestNoHashJoinHint(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("create table t1(a int, b int, key(a));")
tk.MustExec("create table t2(a int, b int, key(a));")
tk.MustExec("create table t3(a int, b int, key(a));")
tk.MustExec("create table t4(a int, b int, key(a));")
runJoinReorderTestData(t, tk, "TestNoHashJoinHint")
}

func TestLeadingJoinHint(t *testing.T) {
store := testkit.CreateMockStore(t)

Expand Down
85 changes: 19 additions & 66 deletions planner/core/casetest/rule/testdata/join_reorder_suite_in.json

Large diffs are not rendered by default.

199 changes: 199 additions & 0 deletions planner/core/casetest/rule/testdata/join_reorder_suite_out.json
Original file line number Diff line number Diff line change
Expand Up @@ -583,6 +583,205 @@
}
]
},
{
"Name": "TestNoHashJoinHint",
"Cases": [
{
"SQL": "select /*+ no_hash_join() */ * from t1, t2",
"Plan": [
"HashJoin 100000000.00 root CARTESIAN inner join",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": [
"Warning 1815 Hint no_hash_join() is inapplicable. Please specify the table names in the arguments."
]
},
{
"SQL": "select /*+ no_hash_join(t1), hash_join(t1) */ * from t1, t2",
"Plan": [
"HashJoin 100000000.00 root CARTESIAN inner join",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": [
"Warning 1815 Some HASH_JOIN and NO_HASH_JOIN hints conflict, NO_HASH_JOIN is ignored"
]
},
{
"SQL": "select /*+ no_hash_join(t1), hash_join(t2) */ * from t1, t2",
"Plan": [
"HashJoin 100000000.00 root CARTESIAN inner join",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": [
"Warning 1815 Some HASH_JOIN and NO_HASH_JOIN hints conflict, NO_HASH_JOIN is ignored"
]
},
{
"SQL": "select /*+ no_hash_join(t1) */ * from t1, t2",
"Plan": [
"MergeJoin 100000000.00 root inner join",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t1, t2) */ * from t1, t2",
"Plan": [
"MergeJoin 100000000.00 root inner join",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t1) */ * from t1, t2 where t1.a=t2.a",
"Plan": [
"IndexHashJoin 12487.50 root inner join, inner:IndexLookUp, outer key:test.t1.a, inner key:test.t2.a, equal cond:eq(test.t1.a, test.t2.a)",
"├─TableReader(Build) 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tikv] not(isnull(test.t1.a))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo",
"└─IndexLookUp(Probe) 12487.50 root ",
" ├─Selection(Build) 12487.50 cop[tikv] not(isnull(test.t2.a))",
" │ └─IndexRangeScan 12500.00 cop[tikv] table:t2, index:a(a) range: decided by [eq(test.t2.a, test.t1.a)], keep order:false, stats:pseudo",
" └─TableRowIDScan(Probe) 12487.50 cop[tikv] table:t2 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t1, t2) */ * from t1, t2 where t1.b=t2.b",
"Plan": [
"MergeJoin 12487.50 root inner join, left key:test.t1.b, right key:test.t2.b",
"├─Sort(Build) 9990.00 root test.t2.b",
"│ └─TableReader 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tikv] not(isnull(test.t2.b))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─Sort(Probe) 9990.00 root test.t1.b",
" └─TableReader 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t1.b))",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t1) */ * from t1, t2 where t1.a=t2.a and t1.b=t2.b",
"Plan": [
"IndexHashJoin 12475.01 root inner join, inner:IndexLookUp, outer key:test.t1.a, inner key:test.t2.a, equal cond:eq(test.t1.a, test.t2.a), eq(test.t1.b, test.t2.b)",
"├─TableReader(Build) 9980.01 root data:Selection",
"│ └─Selection 9980.01 cop[tikv] not(isnull(test.t1.a)), not(isnull(test.t1.b))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo",
"└─IndexLookUp(Probe) 12475.01 root ",
" ├─Selection(Build) 12487.50 cop[tikv] not(isnull(test.t2.a))",
" │ └─IndexRangeScan 12500.00 cop[tikv] table:t2, index:a(a) range: decided by [eq(test.t2.a, test.t1.a)], keep order:false, stats:pseudo",
" └─Selection(Probe) 12475.01 cop[tikv] not(isnull(test.t2.b))",
" └─TableRowIDScan 12487.50 cop[tikv] table:t2 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t2) */ * from t1 left join t2 on t1.b=t2.b",
"Plan": [
"MergeJoin 12487.50 root left outer join, left key:test.t1.b, right key:test.t2.b",
"├─Sort(Build) 9990.00 root test.t2.b",
"│ └─TableReader 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tikv] not(isnull(test.t2.b))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─Sort(Probe) 10000.00 root test.t1.b",
" └─TableReader 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t2) */ * from t1 left join t2 on t1.a=t2.a",
"Plan": [
"IndexHashJoin 12487.50 root left outer join, inner:IndexLookUp, outer key:test.t1.a, inner key:test.t2.a, equal cond:eq(test.t1.a, test.t2.a)",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo",
"└─IndexLookUp(Probe) 12487.50 root ",
" ├─Selection(Build) 12487.50 cop[tikv] not(isnull(test.t2.a))",
" │ └─IndexRangeScan 12500.00 cop[tikv] table:t2, index:a(a) range: decided by [eq(test.t2.a, test.t1.a)], keep order:false, stats:pseudo",
" └─TableRowIDScan(Probe) 12487.50 cop[tikv] table:t2 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t2) */ * from t1 right join t2 on t1.b=t2.b",
"Plan": [
"MergeJoin 12487.50 root right outer join, left key:test.t1.b, right key:test.t2.b",
"├─Sort(Build) 9990.00 root test.t1.b",
"│ └─TableReader 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tikv] not(isnull(test.t1.b))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo",
"└─Sort(Probe) 10000.00 root test.t2.b",
" └─TableReader 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ no_hash_join(t2) */ * from t1 right join t2 on t1.a=t2.a",
"Plan": [
"IndexHashJoin 12487.50 root right outer join, inner:IndexLookUp, outer key:test.t2.a, inner key:test.t1.a, equal cond:eq(test.t2.a, test.t1.a)",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
"└─IndexLookUp(Probe) 12487.50 root ",
" ├─Selection(Build) 12487.50 cop[tikv] not(isnull(test.t1.a))",
" │ └─IndexRangeScan 12500.00 cop[tikv] table:t1, index:a(a) range: decided by [eq(test.t1.a, test.t2.a)], keep order:false, stats:pseudo",
" └─TableRowIDScan(Probe) 12487.50 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ leading(t4, t3, t2, t1), no_hash_join(t2, t3) */ * from t1, t2, t3, t4",
"Plan": [
"Projection 10000000000000000.00 root test.t1.a, test.t1.b, test.t2.a, test.t2.b, test.t3.a, test.t3.b, test.t4.a, test.t4.b",
"└─HashJoin 10000000000000000.00 root CARTESIAN inner join",
" ├─TableReader(Build) 10000.00 root data:TableFullScan",
" │ └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo",
" └─MergeJoin(Probe) 1000000000000.00 root inner join",
" ├─TableReader(Build) 10000.00 root data:TableFullScan",
" │ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
" └─MergeJoin(Probe) 100000000.00 root inner join",
" ├─TableReader(Build) 10000.00 root data:TableFullScan",
" │ └─TableFullScan 10000.00 cop[tikv] table:t3 keep order:false, stats:pseudo",
" └─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t4 keep order:false, stats:pseudo"
],
"Warning": null
},
{
"SQL": "select /*+ leading(t1, t2, t3, t4), hash_join(t1, t2), no_hash_join(t3), hash_join(t4) */ * from t1, t2, t3, t4",
"Plan": [
"HashJoin 10000000000000000.00 root CARTESIAN inner join",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t4 keep order:false, stats:pseudo",
"└─MergeJoin(Probe) 1000000000000.00 root inner join",
" ├─TableReader(Build) 10000.00 root data:TableFullScan",
" │ └─TableFullScan 10000.00 cop[tikv] table:t3 keep order:false, stats:pseudo",
" └─HashJoin(Probe) 100000000.00 root CARTESIAN inner join",
" ├─TableReader(Build) 10000.00 root data:TableFullScan",
" │ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
" └─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
],
"Warning": null
}
]
},
{
"Name": "TestLeadingJoinHint",
"Cases": [
Expand Down
12 changes: 11 additions & 1 deletion planner/core/exhaust_physical_plans.go
Original file line number Diff line number Diff line change
Expand Up @@ -233,7 +233,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 @@ -391,6 +392,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 @@ -437,7 +439,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 @@ -98,6 +98,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 @@ -745,6 +747,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 @@ -3914,6 +3924,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 @@ -3928,7 +3939,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 @@ -3951,6 +3962,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 @@ -4061,6 +4074,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 @@ -7305,6 +7319,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 @@ -122,6 +122,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 @@ -96,6 +96,7 @@ type tableHintInfo struct {
broadcastJoinTables []hintTableInfo
shuffleJoinTables []hintTableInfo
hashJoinTables []hintTableInfo
noHashJoinTables []hintTableInfo
indexHintList []indexHintInfo
tiflashTables []hintTableInfo
tikvTables []hintTableInfo
Expand Down Expand Up @@ -240,6 +241,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

0 comments on commit 7c2dbbe

Please sign in to comment.