From feaa6d59d98a265d0c8a457e2015c849dc644ca5 Mon Sep 17 00:00:00 2001 From: Harshit Gangal Date: Mon, 28 Oct 2024 10:02:13 +0530 Subject: [PATCH] fix: use related semtable for each operator to sql builder conversion Signed-off-by: Harshit Gangal --- .../planbuilder/operator_transformers.go | 59 ++++++------ .../planbuilder/operators/SQL_builder.go | 13 +-- .../vtgate/planbuilder/operators/ast_to_op.go | 15 ++- go/vt/vtgate/planbuilder/operators/delete.go | 7 +- .../planbuilder/operators/expressions.go | 5 +- .../planbuilder/operators/fk_cascade.go | 5 + .../vtgate/planbuilder/operators/fk_verify.go | 5 +- go/vt/vtgate/planbuilder/operators/update.go | 29 +++--- .../plancontext/planning_context.go | 94 +++++++++++++++++-- .../testdata/foreignkey_cases.json | 4 +- .../planbuilder/testdata/from_cases.json | 4 +- .../testdata/postprocess_cases.json | 2 +- .../planbuilder/testdata/reference_cases.json | 2 +- .../planbuilder/testdata/select_cases.json | 4 +- .../planbuilder/testdata/tpcc_cases.json | 4 +- 15 files changed, 172 insertions(+), 80 deletions(-) diff --git a/go/vt/vtgate/planbuilder/operator_transformers.go b/go/vt/vtgate/planbuilder/operator_transformers.go index 127404dee9f..5a9574dfb79 100644 --- a/go/vt/vtgate/planbuilder/operator_transformers.go +++ b/go/vt/vtgate/planbuilder/operator_transformers.go @@ -79,9 +79,8 @@ func transformFkCascade(ctx *plancontext.PlanningContext, fkc *operators.FkCasca return nil, nil } - // Once we have the parent logical plan, we can create the selection logical plan and the primitives for the children operators. - // For all of these, we don't need the semTable anymore. We set it to nil, to avoid using an incorrect one. - ctx.SemTable = nil + // Using the correct semantics.SemTable for the selection query created during planning. + ctx.SemTable = fkc.SSemTable selLP, err := transformToLogicalPlan(ctx, fkc.Selection) if err != nil { return nil, err @@ -90,6 +89,8 @@ func transformFkCascade(ctx *plancontext.PlanningContext, fkc *operators.FkCasca // Go over the children and convert them to Primitives too. var children []*engine.FkChild for _, child := range fkc.Children { + // Using the correct semantics.SemTable for the child table cascade query created during planning. + ctx.SemTable = child.ST childLP, err := transformToLogicalPlan(ctx, child.Op) if err != nil { return nil, err @@ -109,6 +110,31 @@ func transformFkCascade(ctx *plancontext.PlanningContext, fkc *operators.FkCasca return newFkCascade(parentLP, selLP, children), nil } +// transformFkVerify transforms a FkVerify operator into a logical plan. +func transformFkVerify(ctx *plancontext.PlanningContext, fkv *operators.FkVerify) (logicalPlan, error) { + inputLP, err := transformToLogicalPlan(ctx, fkv.Input) + if err != nil { + return nil, err + } + + // Go over the children and convert them to Primitives too. + var verify []*verifyLP + for _, v := range fkv.Verify { + // Using the correct semantics.SemTable for the parent table verify query created during planning. + ctx.SemTable = v.ST + lp, err := transformToLogicalPlan(ctx, v.Op) + if err != nil { + return nil, err + } + verify = append(verify, &verifyLP{ + verify: lp, + typ: v.Typ, + }) + } + + return newFkVerify(inputLP, verify), nil +} + func transformSubQuery(ctx *plancontext.PlanningContext, op *operators.SubQuery) (logicalPlan, error) { outer, err := transformToLogicalPlan(ctx, op.Outer) if err != nil { @@ -136,33 +162,6 @@ func transformSubQuery(ctx *plancontext.PlanningContext, op *operators.SubQuery) return newSemiJoin(outer, inner, op.Vars, lhsCols), nil } -// transformFkVerify transforms a FkVerify operator into a logical plan. -func transformFkVerify(ctx *plancontext.PlanningContext, fkv *operators.FkVerify) (logicalPlan, error) { - inputLP, err := transformToLogicalPlan(ctx, fkv.Input) - if err != nil { - return nil, err - } - - // Once we have the input logical plan, we can create the primitives for the verification operators. - // For all of these, we don't need the semTable anymore. We set it to nil, to avoid using an incorrect one. - ctx.SemTable = nil - - // Go over the children and convert them to Primitives too. - var verify []*verifyLP - for _, v := range fkv.Verify { - lp, err := transformToLogicalPlan(ctx, v.Op) - if err != nil { - return nil, err - } - verify = append(verify, &verifyLP{ - verify: lp, - typ: v.Typ, - }) - } - - return newFkVerify(inputLP, verify), nil -} - func transformAggregator(ctx *plancontext.PlanningContext, op *operators.Aggregator) (logicalPlan, error) { plan, err := transformToLogicalPlan(ctx, op.Source) if err != nil { diff --git a/go/vt/vtgate/planbuilder/operators/SQL_builder.go b/go/vt/vtgate/planbuilder/operators/SQL_builder.go index cd56fed05b2..e2ccf41d028 100644 --- a/go/vt/vtgate/planbuilder/operators/SQL_builder.go +++ b/go/vt/vtgate/planbuilder/operators/SQL_builder.go @@ -88,7 +88,7 @@ func (qb *queryBuilder) addTableExpr( } func (qb *queryBuilder) addPredicate(expr sqlparser.Expr) { - if _, toBeSkipped := qb.ctx.SkipPredicates[expr]; toBeSkipped { + if qb.ctx.ShouldSkip(expr) { // This is a predicate that was added to the RHS of an ApplyJoin. // The original predicate will be added, so we don't have to add this here return @@ -566,21 +566,16 @@ func buildProjection(op *Projection, qb *queryBuilder) error { func buildApplyJoin(op *ApplyJoin, qb *queryBuilder) error { predicates := slice.Map(op.JoinPredicates, func(jc JoinColumn) sqlparser.Expr { // since we are adding these join predicates, we need to mark to broken up version (RHSExpr) of it as done - qb.ctx.SkipPredicates[jc.RHSExpr] = nil - + _ = qb.ctx.SkipJoinPredicates(jc.Original.Expr) return jc.Original.Expr }) + pred := sqlparser.AndExpressions(predicates...) err := buildQuery(op.LHS, qb) if err != nil { return err } - // If we are going to add the predicate used in join here - // We should not add the predicate's copy of when it was split into - // two parts. To avoid this, we use the SkipPredicates map. - for _, pred := range op.JoinPredicates { - qb.ctx.SkipPredicates[pred.RHSExpr] = nil - } + qbR := &queryBuilder{ctx: qb.ctx} err = buildQuery(op.RHS, qbR) if err != nil { diff --git a/go/vt/vtgate/planbuilder/operators/ast_to_op.go b/go/vt/vtgate/planbuilder/operators/ast_to_op.go index e7628edacc5..4bb1e5d91d6 100644 --- a/go/vt/vtgate/planbuilder/operators/ast_to_op.go +++ b/go/vt/vtgate/planbuilder/operators/ast_to_op.go @@ -204,16 +204,23 @@ func createOperatorFromUnion(ctx *plancontext.PlanningContext, node *sqlparser.U // createOpFromStmt creates an operator from the given statement. It takes in two additional arguments— // 1. verifyAllFKs: For this given statement, do we need to verify validity of all the foreign keys on the vtgate level. // 2. fkToIgnore: The foreign key constraint to specifically ignore while planning the statement. -func createOpFromStmt(ctx *plancontext.PlanningContext, stmt sqlparser.Statement, verifyAllFKs bool, fkToIgnore string) (ops.Operator, error) { +func createOpFromStmt(ctx *plancontext.PlanningContext, stmt sqlparser.Statement, verifyAllFKs bool, fkToIgnore string) (*semantics.SemTable, ops.Operator, error) { newCtx, err := plancontext.CreatePlanningContext(stmt, ctx.ReservedVars, ctx.VSchema, ctx.PlannerVersion) if err != nil { - return nil, err + return nil, nil, err } newCtx.VerifyAllFKs = verifyAllFKs newCtx.ParentFKToIgnore = fkToIgnore - return PlanQuery(newCtx, stmt) + query, err := PlanQuery(newCtx, stmt) + if err != nil { + return nil, nil, err + } + + ctx.KeepPredicateInfo(newCtx) + + return newCtx.SemTable, query, err } func getOperatorFromTableExpr(ctx *plancontext.PlanningContext, tableExpr sqlparser.TableExpr, onlyTable bool) (ops.Operator, error) { @@ -377,7 +384,7 @@ func createSelectionOp( where *sqlparser.Where, limit *sqlparser.Limit, lock sqlparser.Lock, -) (ops.Operator, error) { +) (*semantics.SemTable, ops.Operator, error) { selectionStmt := &sqlparser.Select{ SelectExprs: selectExprs, From: tableExprs, diff --git a/go/vt/vtgate/planbuilder/operators/delete.go b/go/vt/vtgate/planbuilder/operators/delete.go index 11a4ed8b31d..ad265101d01 100644 --- a/go/vt/vtgate/planbuilder/operators/delete.go +++ b/go/vt/vtgate/planbuilder/operators/delete.go @@ -195,7 +195,7 @@ func createFkCascadeOpForDelete(ctx *plancontext.PlanningContext, parentOp ops.O } fkChildren = append(fkChildren, fkChild) } - selectionOp, err := createSelectionOp(ctx, selectExprs, delStmt.TableExprs, delStmt.Where, nil, sqlparser.ForUpdateLock) + st, selectionOp, err := createSelectionOp(ctx, selectExprs, delStmt.TableExprs, delStmt.Where, nil, sqlparser.ForUpdateLock) if err != nil { return nil, err } @@ -204,6 +204,8 @@ func createFkCascadeOpForDelete(ctx *plancontext.PlanningContext, parentOp ops.O Selection: selectionOp, Children: fkChildren, Parent: parentOp, + + SSemTable: st, }, nil } @@ -247,7 +249,7 @@ func createFkChildForDelete(ctx *plancontext.PlanningContext, fk vindexes.ChildF } // For the child statement of a DELETE query, we don't need to verify all the FKs on VTgate or ignore any foreign key explicitly. - childOp, err := createOpFromStmt(ctx, childStmt, false /* verifyAllFKs */, "" /* fkToIgnore */) + cST, childOp, err := createOpFromStmt(ctx, childStmt, false /* verifyAllFKs */, "" /* fkToIgnore */) if err != nil { return nil, err } @@ -256,5 +258,6 @@ func createFkChildForDelete(ctx *plancontext.PlanningContext, fk vindexes.ChildF BVName: bvName, Cols: cols, Op: childOp, + ST: cST, }, nil } diff --git a/go/vt/vtgate/planbuilder/operators/expressions.go b/go/vt/vtgate/planbuilder/operators/expressions.go index 4c03490317e..defe2e506ad 100644 --- a/go/vt/vtgate/planbuilder/operators/expressions.go +++ b/go/vt/vtgate/planbuilder/operators/expressions.go @@ -52,10 +52,7 @@ func BreakExpressionInLHSandRHS( cursor.Replace(arg) }, nil).(sqlparser.Expr) - if err != nil { - return JoinColumn{}, err - } - ctx.JoinPredicates[expr] = append(ctx.JoinPredicates[expr], rewrittenExpr) + ctx.AddJoinPredicates(expr, rewrittenExpr) col.RHSExpr = rewrittenExpr return } diff --git a/go/vt/vtgate/planbuilder/operators/fk_cascade.go b/go/vt/vtgate/planbuilder/operators/fk_cascade.go index a9afbde0a7c..40ca29507e1 100644 --- a/go/vt/vtgate/planbuilder/operators/fk_cascade.go +++ b/go/vt/vtgate/planbuilder/operators/fk_cascade.go @@ -20,6 +20,7 @@ import ( "slices" "vitess.io/vitess/go/vt/vtgate/planbuilder/operators/ops" + "vitess.io/vitess/go/vt/vtgate/semantics" ) // FkChild is used to represent a foreign key child table operation @@ -27,6 +28,7 @@ type FkChild struct { BVName string Cols []int // indexes Op ops.Operator + ST *semantics.SemTable noColumns noPredicates @@ -37,6 +39,7 @@ type FkChild struct { // cascades (for example, ON DELETE CASCADE). type FkCascade struct { Selection ops.Operator + SSemTable *semantics.SemTable Children []*FkChild Parent ops.Operator @@ -80,6 +83,7 @@ func (fkc *FkCascade) Clone(inputs []ops.Operator) ops.Operator { newFkc := &FkCascade{ Parent: inputs[0], Selection: inputs[1], + SSemTable: fkc.SSemTable, } for idx, operator := range inputs { if idx < 2 { @@ -90,6 +94,7 @@ func (fkc *FkCascade) Clone(inputs []ops.Operator) ops.Operator { BVName: fkc.Children[idx-2].BVName, Cols: slices.Clone(fkc.Children[idx-2].Cols), Op: operator, + ST: fkc.Children[idx-2].ST, }) } return newFkc diff --git a/go/vt/vtgate/planbuilder/operators/fk_verify.go b/go/vt/vtgate/planbuilder/operators/fk_verify.go index 8c2431d26fc..27212a2c67a 100644 --- a/go/vt/vtgate/planbuilder/operators/fk_verify.go +++ b/go/vt/vtgate/planbuilder/operators/fk_verify.go @@ -18,12 +18,15 @@ package operators import ( "vitess.io/vitess/go/vt/vtgate/planbuilder/operators/ops" + "vitess.io/vitess/go/vt/vtgate/semantics" ) // VerifyOp keeps the information about the foreign key verification operation. // It is a Parent verification or a Child verification. type VerifyOp struct { - Op ops.Operator + Op ops.Operator + ST *semantics.SemTable + Typ string } diff --git a/go/vt/vtgate/planbuilder/operators/update.go b/go/vt/vtgate/planbuilder/operators/update.go index 5a7716bdbeb..40ac5820dd1 100644 --- a/go/vt/vtgate/planbuilder/operators/update.go +++ b/go/vt/vtgate/planbuilder/operators/update.go @@ -344,13 +344,14 @@ func createFKCascadeOp(ctx *plancontext.PlanningContext, parentOp ops.Operator, fkChildren = append(fkChildren, fkChild) } - selectionOp, err := createSelectionOp(ctx, selectExprs, updStmt.TableExprs, updStmt.Where, nil, sqlparser.ForUpdateLock) + st, selectionOp, err := createSelectionOp(ctx, selectExprs, updStmt.TableExprs, updStmt.Where, nil, sqlparser.ForUpdateLock) if err != nil { return nil, err } return &FkCascade{ Selection: selectionOp, + SSemTable: st, Children: fkChildren, Parent: parentOp, }, nil @@ -370,13 +371,14 @@ func createFkChildForUpdate(ctx *plancontext.PlanningContext, fk vindexes.ChildF compExpr := sqlparser.NewComparisonExpr(sqlparser.InOp, valTuple, sqlparser.NewListArg(bvName), nil) var childWhereExpr sqlparser.Expr = compExpr + var st *semantics.SemTable var childOp ops.Operator var err error switch fk.OnUpdate { case sqlparser.Cascade: - childOp, err = buildChildUpdOpForCascade(ctx, fk, updStmt, childWhereExpr, updatedTable) + st, childOp, err = buildChildUpdOpForCascade(ctx, fk, updStmt, childWhereExpr, updatedTable) case sqlparser.SetNull: - childOp, err = buildChildUpdOpForSetNull(ctx, fk, updStmt, childWhereExpr) + st, childOp, err = buildChildUpdOpForSetNull(ctx, fk, updStmt, childWhereExpr) case sqlparser.SetDefault: return nil, vterrors.VT09016() } @@ -388,6 +390,7 @@ func createFkChildForUpdate(ctx *plancontext.PlanningContext, fk vindexes.ChildF BVName: bvName, Cols: cols, Op: childOp, + ST: st, }, nil } @@ -395,7 +398,7 @@ func createFkChildForUpdate(ctx *plancontext.PlanningContext, fk vindexes.ChildF // The query looks like this - // // `UPDATE SET WHERE IN ()` -func buildChildUpdOpForCascade(ctx *plancontext.PlanningContext, fk vindexes.ChildFKInfo, updStmt *sqlparser.Update, childWhereExpr sqlparser.Expr, updatedTable *vindexes.Table) (ops.Operator, error) { +func buildChildUpdOpForCascade(ctx *plancontext.PlanningContext, fk vindexes.ChildFKInfo, updStmt *sqlparser.Update, childWhereExpr sqlparser.Expr, updatedTable *vindexes.Table) (*semantics.SemTable, ops.Operator, error) { // The update expressions are the same as the update expressions in the parent update query // with the column names replaced with the child column names. var childUpdateExprs sqlparser.UpdateExprs @@ -436,7 +439,7 @@ func buildChildUpdOpForCascade(ctx *plancontext.PlanningContext, fk vindexes.Chi // `UPDATE SET // WHERE IN () // [AND ({ IS NULL OR}... NOT IN ())]` -func buildChildUpdOpForSetNull(ctx *plancontext.PlanningContext, fk vindexes.ChildFKInfo, updStmt *sqlparser.Update, childWhereExpr sqlparser.Expr) (ops.Operator, error) { +func buildChildUpdOpForSetNull(ctx *plancontext.PlanningContext, fk vindexes.ChildFKInfo, updStmt *sqlparser.Update, childWhereExpr sqlparser.Expr) (*semantics.SemTable, ops.Operator, error) { // For the SET NULL type constraint, we need to set all the child columns to NULL. var childUpdateExprs sqlparser.UpdateExprs for _, column := range fk.ChildColumns { @@ -479,23 +482,25 @@ func createFKVerifyOp(ctx *plancontext.PlanningContext, childOp ops.Operator, up var Verify []*VerifyOp // This validates that new values exists on the parent table. for _, fk := range parentFks { - op, err := createFkVerifyOpForParentFKForUpdate(ctx, updStmt, fk) + st, op, err := createFkVerifyOpForParentFKForUpdate(ctx, updStmt, fk) if err != nil { return nil, err } Verify = append(Verify, &VerifyOp{ Op: op, + ST: st, Typ: engine.ParentVerify, }) } // This validates that the old values don't exist on the child table. for _, fk := range restrictChildFks { - op, err := createFkVerifyOpForChildFKForUpdate(ctx, updStmt, fk) + st, op, err := createFkVerifyOpForChildFKForUpdate(ctx, updStmt, fk) if err != nil { return nil, err } Verify = append(Verify, &VerifyOp{ Op: op, + ST: st, Typ: engine.ChildVerify, }) } @@ -517,11 +522,11 @@ func createFKVerifyOp(ctx *plancontext.PlanningContext, childOp ops.Operator, up // where Parent.p1 is null and Parent.p2 is null and Child.id = 1 // and Child.c2 is not null // limit 1 -func createFkVerifyOpForParentFKForUpdate(ctx *plancontext.PlanningContext, updStmt *sqlparser.Update, pFK vindexes.ParentFKInfo) (ops.Operator, error) { +func createFkVerifyOpForParentFKForUpdate(ctx *plancontext.PlanningContext, updStmt *sqlparser.Update, pFK vindexes.ParentFKInfo) (*semantics.SemTable, ops.Operator, error) { childTblExpr := updStmt.TableExprs[0].(*sqlparser.AliasedTableExpr) childTbl, err := childTblExpr.TableName() if err != nil { - return nil, err + return nil, nil, err } parentTbl := pFK.Table.GetTableName() var whereCond sqlparser.Expr @@ -594,16 +599,16 @@ func createFkVerifyOpForParentFKForUpdate(ctx *plancontext.PlanningContext, updS // verify query: // select 1 from Child join Parent on Parent.p1 = Child.c1 and Parent.p2 = Child.c2 // where Parent.id = 1 and (1 IS NULL OR (child.c1) NOT IN ((1))) limit 1 -func createFkVerifyOpForChildFKForUpdate(ctx *plancontext.PlanningContext, updStmt *sqlparser.Update, cFk vindexes.ChildFKInfo) (ops.Operator, error) { +func createFkVerifyOpForChildFKForUpdate(ctx *plancontext.PlanningContext, updStmt *sqlparser.Update, cFk vindexes.ChildFKInfo) (*semantics.SemTable, ops.Operator, error) { // ON UPDATE RESTRICT foreign keys that require validation, should only be allowed in the case where we // are verifying all the FKs on vtgate level. if !ctx.VerifyAllFKs { - return nil, vterrors.VT12002() + return nil, nil, vterrors.VT12002() } parentTblExpr := updStmt.TableExprs[0].(*sqlparser.AliasedTableExpr) parentTbl, err := parentTblExpr.TableName() if err != nil { - return nil, err + return nil, nil, err } childTbl := cFk.Table.GetTableName() var joinCond sqlparser.Expr diff --git a/go/vt/vtgate/planbuilder/plancontext/planning_context.go b/go/vt/vtgate/planbuilder/plancontext/planning_context.go index d090a593a39..d9ebb95ff85 100644 --- a/go/vt/vtgate/planbuilder/plancontext/planning_context.go +++ b/go/vt/vtgate/planbuilder/plancontext/planning_context.go @@ -19,6 +19,7 @@ package plancontext import ( querypb "vitess.io/vitess/go/vt/proto/query" "vitess.io/vitess/go/vt/sqlparser" + "vitess.io/vitess/go/vt/vterrors" "vitess.io/vitess/go/vt/vtgate/semantics" ) @@ -27,12 +28,16 @@ type PlanningContext struct { SemTable *semantics.SemTable VSchema VSchema - // here we add all predicates that were created because of a join condition - // e.g. [FROM tblA JOIN tblB ON a.colA = b.colB] will be rewritten to [FROM tblB WHERE :a_colA = b.colB], - // if we assume that tblB is on the RHS of the join. This last predicate in the WHERE clause is added to the - // map below - JoinPredicates map[sqlparser.Expr][]sqlparser.Expr - SkipPredicates map[sqlparser.Expr]any + // joinPredicates maps each original join predicate (key) to a slice of + // variations of the RHS predicates (value). This map is used to handle + // different scenarios in join planning, where the RHS predicates are + // modified to accommodate dependencies + joinPredicates map[sqlparser.Expr][]sqlparser.Expr + + // skipPredicates tracks predicates that should be skipped, typically when + // a join predicate is reverted to its original form during planning. + skipPredicates map[sqlparser.Expr]any + PlannerVersion querypb.ExecuteOptions_PlannerVersion // If we during planning have turned this expression into an argument name, @@ -79,8 +84,8 @@ func CreatePlanningContext(stmt sqlparser.Statement, ReservedVars: reservedVars, SemTable: semTable, VSchema: vschema, - JoinPredicates: map[sqlparser.Expr][]sqlparser.Expr{}, - SkipPredicates: map[sqlparser.Expr]any{}, + joinPredicates: map[sqlparser.Expr][]sqlparser.Expr{}, + skipPredicates: map[sqlparser.Expr]any{}, PlannerVersion: version, ReservedArguments: map[sqlparser.Expr]string{}, }, nil @@ -116,3 +121,76 @@ func (ctx *PlanningContext) GetArgumentFor(expr sqlparser.Expr, f func() string) ctx.ReservedArguments[expr] = bvName return bvName } + +// ShouldSkip determines if a given expression should be ignored in the SQL output building. +// It checks against expressions that have been marked to be excluded from further processing. +func (ctx *PlanningContext) ShouldSkip(expr sqlparser.Expr) bool { + for k := range ctx.skipPredicates { + if ctx.SemTable.EqualsExpr(expr, k) { + return true + } + } + return false +} + +// AddJoinPredicates associates additional RHS predicates with an existing join predicate. +// This is used to dynamically adjust the RHS predicates based on evolving join conditions. +func (ctx *PlanningContext) AddJoinPredicates(joinPred sqlparser.Expr, predicates ...sqlparser.Expr) { + fn := func(original sqlparser.Expr, rhsExprs []sqlparser.Expr) { + ctx.joinPredicates[original] = append(rhsExprs, predicates...) + } + if ctx.execOnJoinPredicateEqual(joinPred, fn) { + return + } + + // we didn't find an existing entry + ctx.joinPredicates[joinPred] = predicates +} + +// SkipJoinPredicates marks the predicates related to a specific join predicate as irrelevant +// for the current planning stage. This is used when a join has been pushed under a route and +// the original predicate will be used. +func (ctx *PlanningContext) SkipJoinPredicates(joinPred sqlparser.Expr) error { + fn := func(_ sqlparser.Expr, rhsExprs []sqlparser.Expr) { + ctx.skipThesePredicates(rhsExprs...) + } + if ctx.execOnJoinPredicateEqual(joinPred, fn) { + return nil + } + return vterrors.VT13001("predicate does not exist: " + sqlparser.String(joinPred)) +} + +// KeepPredicateInfo transfers join predicate information from another context. +// This is useful when nesting queries, ensuring consistent predicate handling across contexts. +func (ctx *PlanningContext) KeepPredicateInfo(other *PlanningContext) { + for k, v := range other.joinPredicates { + ctx.AddJoinPredicates(k, v...) + } + for expr := range other.skipPredicates { + ctx.skipThesePredicates(expr) + } +} + +// skipThesePredicates is a utility function to exclude certain predicates from SQL building +func (ctx *PlanningContext) skipThesePredicates(preds ...sqlparser.Expr) { +outer: + for _, expr := range preds { + for k := range ctx.skipPredicates { + if ctx.SemTable.EqualsExpr(expr, k) { + // already skipped + continue outer + } + } + ctx.skipPredicates[expr] = nil + } +} + +func (ctx *PlanningContext) execOnJoinPredicateEqual(joinPred sqlparser.Expr, fn func(original sqlparser.Expr, rhsExprs []sqlparser.Expr)) bool { + for key, values := range ctx.joinPredicates { + if ctx.SemTable.EqualsExpr(joinPred, key) { + fn(key, values) + return true + } + } + return false +} diff --git a/go/vt/vtgate/planbuilder/testdata/foreignkey_cases.json b/go/vt/vtgate/planbuilder/testdata/foreignkey_cases.json index c9c0acb3cc7..2cc4bf4d4a3 100644 --- a/go/vt/vtgate/planbuilder/testdata/foreignkey_cases.json +++ b/go/vt/vtgate/planbuilder/testdata/foreignkey_cases.json @@ -1220,7 +1220,7 @@ "Sharded": false }, "FieldQuery": "select 1 from u_tbl4, u_tbl9 where 1 != 1", - "Query": "select 1 from u_tbl4, u_tbl9 where u_tbl4.col4 = u_tbl9.col9 and (u_tbl4.col4) in ::fkc_vals and ('foo' is null or (u_tbl9.col9) not in (('foo'))) and u_tbl4.col4 = u_tbl9.col9 and (u_tbl4.col4) in ::fkc_vals and ('foo' is null or (u_tbl9.col9) not in (('foo'))) limit 1 lock in share mode", + "Query": "select 1 from u_tbl4, u_tbl9 where u_tbl4.col4 = u_tbl9.col9 and (u_tbl4.col4) in ::fkc_vals and ('foo' is null or (u_tbl9.col9) not in (('foo'))) limit 1 lock in share mode", "Table": "u_tbl4, u_tbl9" }, { @@ -1309,7 +1309,7 @@ "Sharded": false }, "FieldQuery": "select 1 from u_tbl4, u_tbl9 where 1 != 1", - "Query": "select 1 from u_tbl4, u_tbl9 where u_tbl4.col4 = u_tbl9.col9 and (u_tbl4.col4) in ::fkc_vals and (:v1 is null or (u_tbl9.col9) not in ((:v1))) and u_tbl4.col4 = u_tbl9.col9 and (u_tbl4.col4) in ::fkc_vals and (:v1 is null or (u_tbl9.col9) not in ((:v1))) limit 1 lock in share mode", + "Query": "select 1 from u_tbl4, u_tbl9 where u_tbl4.col4 = u_tbl9.col9 and (u_tbl4.col4) in ::fkc_vals and (:v1 is null or (u_tbl9.col9) not in ((:v1))) limit 1 lock in share mode", "Table": "u_tbl4, u_tbl9" }, { diff --git a/go/vt/vtgate/planbuilder/testdata/from_cases.json b/go/vt/vtgate/planbuilder/testdata/from_cases.json index 9e668bd68a2..eb5feecf7f6 100644 --- a/go/vt/vtgate/planbuilder/testdata/from_cases.json +++ b/go/vt/vtgate/planbuilder/testdata/from_cases.json @@ -1598,7 +1598,7 @@ "Sharded": true }, "FieldQuery": "select t.id from (select id from `user` where 1 != 1) as t, user_extra where 1 != 1", - "Query": "select t.id from (select id from `user` where id = 5 and id = :user_extra_user_id) as t, user_extra where t.id = user_extra.user_id", + "Query": "select t.id from (select id from `user` where id = 5) as t, user_extra where t.id = user_extra.user_id", "Table": "`user`, user_extra", "Values": [ "INT64(5)" @@ -1736,7 +1736,7 @@ "Sharded": true }, "FieldQuery": "select t.id from (select id, textcol1 as baz from `user` as route1 where 1 != 1) as t, (select id, textcol1 + textcol1 as baz from `user` where 1 != 1) as s where 1 != 1", - "Query": "select t.id from (select id, textcol1 as baz from `user` as route1 where textcol1 = '3') as t, (select id, textcol1 + textcol1 as baz from `user` where textcol1 + textcol1 = '3' and id = :t_id) as s where t.id = s.id", + "Query": "select t.id from (select id, textcol1 as baz from `user` as route1 where textcol1 = '3') as t, (select id, textcol1 + textcol1 as baz from `user` where textcol1 + textcol1 = '3') as s where t.id = s.id", "Table": "`user`" }, "TablesUsed": [ diff --git a/go/vt/vtgate/planbuilder/testdata/postprocess_cases.json b/go/vt/vtgate/planbuilder/testdata/postprocess_cases.json index 7f573227e65..a36ad580dc4 100644 --- a/go/vt/vtgate/planbuilder/testdata/postprocess_cases.json +++ b/go/vt/vtgate/planbuilder/testdata/postprocess_cases.json @@ -1052,7 +1052,7 @@ "Sharded": true }, "FieldQuery": "select * from (select user_id from user_extra where 1 != 1) as eu, `user` as u where 1 != 1", - "Query": "select * from (select user_id from user_extra where user_id = 5 and user_id = :u_id) as eu, `user` as u where u.id = 5 and u.id = eu.user_id order by eu.user_id asc", + "Query": "select * from (select user_id from user_extra where user_id = 5) as eu, `user` as u where u.id = 5 and u.id = eu.user_id order by eu.user_id asc", "Table": "`user`, user_extra", "Values": [ "INT64(5)" diff --git a/go/vt/vtgate/planbuilder/testdata/reference_cases.json b/go/vt/vtgate/planbuilder/testdata/reference_cases.json index 140ce9f7849..e55eed68678 100644 --- a/go/vt/vtgate/planbuilder/testdata/reference_cases.json +++ b/go/vt/vtgate/planbuilder/testdata/reference_cases.json @@ -937,7 +937,7 @@ "Sharded": true }, "FieldQuery": "select 1 from `user` as u, user_extra as ue, ref_with_source as sr, ref as rr where 1 != 1", - "Query": "select 1 from `user` as u, user_extra as ue, ref_with_source as sr, ref as rr where sr.foo = :ue_foo and rr.bar = sr.bar and u.id = ue.user_id and sr.foo = ue.foo", + "Query": "select 1 from `user` as u, user_extra as ue, ref_with_source as sr, ref as rr where rr.bar = sr.bar and u.id = ue.user_id and sr.foo = ue.foo", "Table": "`user`, ref, ref_with_source, user_extra" }, "TablesUsed": [ diff --git a/go/vt/vtgate/planbuilder/testdata/select_cases.json b/go/vt/vtgate/planbuilder/testdata/select_cases.json index 6c255fd9d89..2412fbaadcf 100644 --- a/go/vt/vtgate/planbuilder/testdata/select_cases.json +++ b/go/vt/vtgate/planbuilder/testdata/select_cases.json @@ -4110,7 +4110,7 @@ "Sharded": true }, "FieldQuery": "select music.id from (select id from music where 1 != 1) as other, music where 1 != 1", - "Query": "select music.id from (select id from music where music.user_id = 5 and id = :music_id) as other, music where other.id = music.id", + "Query": "select music.id from (select id from music where music.user_id = 5) as other, music where other.id = music.id", "Table": "music", "Values": [ "INT64(5)" @@ -4136,7 +4136,7 @@ "Sharded": true }, "FieldQuery": "select music.id from (select id from music where 1 != 1) as other, music where 1 != 1", - "Query": "select music.id from (select id from music where music.user_id in ::__vals and id = :music_id) as other, music where other.id = music.id", + "Query": "select music.id from (select id from music where music.user_id in ::__vals) as other, music where other.id = music.id", "Table": "music", "Values": [ "(INT64(5), INT64(6), INT64(7))" diff --git a/go/vt/vtgate/planbuilder/testdata/tpcc_cases.json b/go/vt/vtgate/planbuilder/testdata/tpcc_cases.json index fa823b0ae59..2677deb2cab 100644 --- a/go/vt/vtgate/planbuilder/testdata/tpcc_cases.json +++ b/go/vt/vtgate/planbuilder/testdata/tpcc_cases.json @@ -13,7 +13,7 @@ "Sharded": true }, "FieldQuery": "select c_discount, c_last, c_credit, w_tax from customer1 as c, warehouse1 as w where 1 != 1", - "Query": "select c_discount, c_last, c_credit, w_tax from customer1 as c, warehouse1 as w where c_w_id = :w_id and c_d_id = 15 and c_id = 10 and w_id = 1 and c_w_id = w_id", + "Query": "select c_discount, c_last, c_credit, w_tax from customer1 as c, warehouse1 as w where c_d_id = 15 and c_id = 10 and w_id = 1 and c_w_id = w_id", "Table": "customer1, warehouse1", "Values": [ "INT64(1)" @@ -947,7 +947,7 @@ "Sharded": true }, "FieldQuery": "select o.o_id, o.o_d_id from (select o_c_id, o_w_id, o_d_id, count(distinct o_w_id), o_id from orders1 where 1 != 1 group by o_c_id, o_d_id, o_w_id) as t, orders1 as o where 1 != 1", - "Query": "select o.o_id, o.o_d_id from (select o_c_id, o_w_id, o_d_id, count(distinct o_w_id), o_id from orders1 where o_w_id = 1 and o_id > 2100 and o_id < 11153 and o_w_id = :o_o_w_id and o_d_id = :o_o_d_id and o_c_id = :o_o_c_id group by o_c_id, o_d_id, o_w_id having count(distinct o_id) > 1 limit 1) as t, orders1 as o where t.o_w_id = o.o_w_id and t.o_d_id = o.o_d_id and t.o_c_id = o.o_c_id limit 1", + "Query": "select o.o_id, o.o_d_id from (select o_c_id, o_w_id, o_d_id, count(distinct o_w_id), o_id from orders1 where o_w_id = 1 and o_id > 2100 and o_id < 11153 group by o_c_id, o_d_id, o_w_id having count(distinct o_id) > 1 limit 1) as t, orders1 as o where t.o_w_id = o.o_w_id and t.o_d_id = o.o_d_id and t.o_c_id = o.o_c_id limit 1", "Table": "orders1", "Values": [ "INT64(1)"