Skip to content

Commit

Permalink
planner: add more test cases for join order hint (#34740)
Browse files Browse the repository at this point in the history
ref #29932
  • Loading branch information
Reminiscent authored May 19, 2022
1 parent d3185cc commit eb90d0d
Show file tree
Hide file tree
Showing 6 changed files with 3,305 additions and 24 deletions.
12 changes: 7 additions & 5 deletions planner/core/logical_plan_builder.go
Original file line number Diff line number Diff line change
Expand Up @@ -3630,12 +3630,14 @@ func (b *PlanBuilder) pushTableHints(hints []*ast.TableOptimizerHint, currentLev
// ignore hints that not implemented
}
}
if leadingHintCnt > 1 {
// If there are more leading hints, all leading hints will be invalid.
if leadingHintCnt > 1 || (leadingHintCnt > 0 && b.ctx.GetSessionVars().StmtCtx.StraightJoinOrder) {
// If there are more leading hints or the straight_join hint existes, all leading hints will be invalid.
leadingJoinOrder = leadingJoinOrder[:0]
// Append warning if there are invalid index names.
errMsg := "We can only use one leading hint at most, when multiple leading hints are used, all leading hints will be invalid"
b.ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack(errMsg))
if leadingHintCnt > 1 {
b.ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack("We can only use one leading hint at most, when multiple leading hints are used, all leading hints will be invalid"))
} else if b.ctx.GetSessionVars().StmtCtx.StraightJoinOrder {
b.ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack("We can only use the straight_join hint, when we use the leading hint and straight_join hint at the same time, all leading hints will be invalid"))
}
}
b.tableHintInfo = append(b.tableHintInfo, tableHintInfo{
sortMergeJoinTables: sortMergeTables,
Expand Down
64 changes: 51 additions & 13 deletions planner/core/rule_join_reorder.go
Original file line number Diff line number Diff line change
Expand Up @@ -34,15 +34,22 @@ import (
// For example: "InnerJoin(InnerJoin(a, b), LeftJoin(c, d))"
// results in a join group {a, b, c, d}.
func extractJoinGroup(p LogicalPlan) (group []LogicalPlan, eqEdges []*expression.ScalarFunction,
otherConds []expression.Expression, joinTypes []JoinType, hintInfo *tableHintInfo, hasOuterJoin bool) {
otherConds []expression.Expression, joinTypes []JoinType, hintInfo []*tableHintInfo, hasOuterJoin bool) {
join, isJoin := p.(*LogicalJoin)
if isJoin && join.preferJoinOrder {
// When there is a leading hint, the hint may not take effect for other reasons.
// For example, the join type is cross join or straight join, or exists the join algorithm hint, etc.
// We need to return the hint information to warn
hintInfo = append(hintInfo, join.hintInfo)
}
if !isJoin || join.preferJoinType > uint(0) || join.StraightJoin ||
(join.JoinType != InnerJoin && join.JoinType != LeftOuterJoin && join.JoinType != RightOuterJoin) ||
((join.JoinType == LeftOuterJoin || join.JoinType == RightOuterJoin) && join.EqualConditions == nil) {
return []LogicalPlan{p}, nil, nil, nil, nil, false
}
if join.preferJoinOrder {
hintInfo = join.hintInfo
if hintInfo != nil {
// The leading hint can not work for some reasons. So clear it in the join node.
join.hintInfo = nil
}
return []LogicalPlan{p}, nil, nil, nil, hintInfo, false
}
hasOuterJoin = hasOuterJoin || (join.JoinType != InnerJoin)
if join.JoinType != RightOuterJoin {
Expand Down Expand Up @@ -75,9 +82,7 @@ func extractJoinGroup(p LogicalPlan) (group []LogicalPlan, eqEdges []*expression
eqEdges = append(eqEdges, lhsEqualConds...)
otherConds = append(otherConds, lhsOtherConds...)
joinTypes = append(joinTypes, lhsJoinTypes...)
if hintInfo == nil && lhsHintInfo != nil {
hintInfo = lhsHintInfo
}
hintInfo = append(hintInfo, lhsHintInfo...)
hasOuterJoin = hasOuterJoin || lhsHasOuterJoin
} else {
group = append(group, join.children[0])
Expand Down Expand Up @@ -113,9 +118,7 @@ func extractJoinGroup(p LogicalPlan) (group []LogicalPlan, eqEdges []*expression
eqEdges = append(eqEdges, rhsEqualConds...)
otherConds = append(otherConds, rhsOtherConds...)
joinTypes = append(joinTypes, rhsJoinTypes...)
if hintInfo == nil && rhsHintInfo != nil {
hintInfo = rhsHintInfo
}
hintInfo = append(hintInfo, rhsHintInfo...)
hasOuterJoin = hasOuterJoin || rhsHasOuterJoin
} else {
group = append(group, join.children[1])
Expand Down Expand Up @@ -181,12 +184,17 @@ func (s *joinReOrderSolver) optimizeRecursive(ctx sessionctx.Context, p LogicalP
joinGroupNum := len(curJoinGroup)
useGreedy := joinGroupNum > ctx.GetSessionVars().TiDBOptJoinReorderThreshold || !isSupportDP

if hintInfo != nil && hintInfo.leadingJoinOrder != nil {
leadingHintInfo, hasDiffLeadingHint := checkAndGenerateLeadingHint(hintInfo)
if hasDiffLeadingHint {
ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack("We can only use one leading hint at most, when multiple leading hints are used, all leading hints will be invalid"))
}

if leadingHintInfo != nil && leadingHintInfo.leadingJoinOrder != nil {
if hasOuterJoin {
ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack("leading hint is inapplicable when we have outer join"))
} else {
if useGreedy {
ok, leftJoinGroup := baseGroupSolver.generateLeadingJoinGroup(curJoinGroup, hintInfo)
ok, leftJoinGroup := baseGroupSolver.generateLeadingJoinGroup(curJoinGroup, leadingHintInfo)
if !ok {
ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack("leading hint is inapplicable, check if the leading hint table is valid"))
} else {
Expand Down Expand Up @@ -234,6 +242,9 @@ func (s *joinReOrderSolver) optimizeRecursive(ctx sessionctx.Context, p LogicalP
}
return p, nil
}
if len(curJoinGroup) == 1 && hintInfo != nil {
ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack("leading hint is inapplicable, check the join type or the join algorithm hint"))
}
newChildren := make([]LogicalPlan, 0, len(p.Children()))
for _, child := range p.Children() {
newChild, err := s.optimizeRecursive(ctx, child, tracer)
Expand All @@ -246,6 +257,33 @@ func (s *joinReOrderSolver) optimizeRecursive(ctx sessionctx.Context, p LogicalP
return p, nil
}

// checkAndGenerateLeadingHint used to check and generate the valid leading hint.
// We are allowed to use at most one leading hint in a join group. When more than one,
// all leading hints in the current join group will be invalid.
// For example: select /*+ leading(t3) */ * from (select /*+ leading(t1) */ t2.b from t1 join t2 on t1.a=t2.a) t4 join t3 on t4.b=t3.b
// The Join Group {t1, t2, t3} contains two leading hints includes leading(t3) and leading(t1).
// Although they are in different query blocks, they are conflicting.
// In addition, the table alias 't4' cannot be recognized because of the join group.
func checkAndGenerateLeadingHint(hintInfo []*tableHintInfo) (*tableHintInfo, bool) {
leadingHintNum := len(hintInfo)
var leadingHintInfo *tableHintInfo
hasDiffLeadingHint := false
if leadingHintNum > 0 {
leadingHintInfo = hintInfo[0]
// One join group has one leading hint at most. Check whether there are different join order hints.
for i := 1; i < leadingHintNum; i++ {
if hintInfo[i] != hintInfo[i-1] {
hasDiffLeadingHint = true
break
}
}
if hasDiffLeadingHint {
leadingHintInfo = nil
}
}
return leadingHintInfo, hasDiffLeadingHint
}

// nolint:structcheck
type baseSingleGroupJoinOrderSolver struct {
ctx sessionctx.Context
Expand Down
8 changes: 8 additions & 0 deletions planner/core/rule_join_reorder_greedy.go
Original file line number Diff line number Diff line change
Expand Up @@ -59,6 +59,8 @@ func (s *joinReorderGreedySolver) solve(joinNodePlans []LogicalPlan, tracer *joi
return s.curJoinGroup[i].cumCost < s.curJoinGroup[j].cumCost
})

// joinNodeNum indicates the number of join nodes except leading join nodes in the current join group
joinNodeNum := len(s.curJoinGroup)
if leadingJoinNodes != nil {
// The leadingJoinNodes should be the first element in the s.curJoinGroup.
// So it can be joined first.
Expand All @@ -71,6 +73,12 @@ func (s *joinReorderGreedySolver) solve(joinNodePlans []LogicalPlan, tracer *joi
if err != nil {
return nil, err
}
if joinNodeNum > 0 && len(s.curJoinGroup) == joinNodeNum {
// Getting here means that there is no join condition between the table used in the leading hint and other tables
// For example: select /*+ leading(t3) */ * from t1 join t2 on t1.a=t2.a cross join t3
// We can not let table t3 join first.
s.ctx.GetSessionVars().StmtCtx.AppendWarning(ErrInternal.GenWithStack("leading hint is inapplicable, check if the leading hint table has join conditions with other tables"))
}
cartesianGroup = append(cartesianGroup, newNode.p)
}

Expand Down
Loading

0 comments on commit eb90d0d

Please sign in to comment.