Documentation ¶
Overview ¶
Package operators contains the operators used to plan queries.
The operators go through a few phases while planning:
- Logical In this first pass, we build an operator tree from the incoming parsed query. It will contain logical joins - we still haven't decided on the join algorithm to use yet. At the leaves, it will contain QueryGraphs - these are the tables in the FROM clause that we can easily do join ordering on. The logical tree will represent the full query, including projections, grouping, ordering and so on.
- Physical Once the logical plan has been fully built, we go bottom up and plan which routes that will be used. During this phase, we will also decide which join algorithms should be used on the vtgate level
- Columns & Aggregation Once we know which queries will be sent to the tablets, we go over the tree and decide which columns each operator should output. At this point, we also do offset lookups, so we know at runtime from which columns in the input table we need to read.
Index ¶
- Constants
- func AddPredicate(join JoinOp, ctx *plancontext.PlanningContext, expr sqlparser.Expr, ...) (ops.Operator, error)
- func BreakExpressionInLHSandRHS(ctx *plancontext.PlanningContext, expr sqlparser.Expr, lhs semantics.TableSet) (bvNames []string, columns []*sqlparser.ColName, rewrittenExpr sqlparser.Expr, ...)
- func CheckValid(op ops.Operator) error
- func Clone(op ops.Operator) ops.Operator
- func Compact(ctx *plancontext.PlanningContext, op ops.Operator) (ops.Operator, error)
- func CompareRefInt(a *int, b *int) bool
- func CostOf(op ops.Operator) (cost int)
- func PlanQuery(ctx *plancontext.PlanningContext, selStmt sqlparser.Statement) (ops.Operator, error)
- func QualifiedIdentifier(ks *vindexes.Keyspace, i sqlparser.IdentifierCS) string
- func QualifiedString(ks *vindexes.Keyspace, s string) string
- func QualifiedStrings(ks *vindexes.Keyspace, ss []string) []string
- func QualifiedTableName(ks *vindexes.Keyspace, t sqlparser.TableName) string
- func QualifiedTableNames(ks *vindexes.Keyspace, ts []sqlparser.TableName) []string
- func QualifiedTables(ks *vindexes.Keyspace, vts []*vindexes.Table) []string
- func RemovePredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr, op ops.Operator) (ops.Operator, error)
- func SingleQualifiedIdentifier(ks *vindexes.Keyspace, i sqlparser.IdentifierCS) []string
- func SingleQualifiedString(ks *vindexes.Keyspace, s string) []string
- func SingleQualifiedTableName(ks *vindexes.Keyspace, t sqlparser.TableName) []string
- func SortAggregations(a []Aggr)
- func SortGrouping(a []GroupBy)
- func TableID(op ops.Operator) (result semantics.TableSet)
- func TablesUsed(op ops.Operator) []string
- func ToSQL(ctx *plancontext.PlanningContext, op ops.Operator) (sqlparser.SelectStatement, error)
- func UnresolvedPredicates(op ops.Operator, st *semantics.SemTable) (result []sqlparser.Expr)
- type Aggr
- type AggrRewriter
- type ApplyJoin
- func (a *ApplyJoin) AddColumn(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
- func (a *ApplyJoin) AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) error
- func (a *ApplyJoin) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (a *ApplyJoin) Clone(inputs []ops.Operator) ops.Operator
- func (a *ApplyJoin) GetLHS() ops.Operator
- func (a *ApplyJoin) GetRHS() ops.Operator
- func (a *ApplyJoin) IPhysical()
- func (a *ApplyJoin) Inputs() []ops.Operator
- func (a *ApplyJoin) IsInner() bool
- func (a *ApplyJoin) MakeInner()
- func (a *ApplyJoin) SetLHS(operator ops.Operator)
- func (a *ApplyJoin) SetRHS(operator ops.Operator)
- type ColNameColumns
- type CorrelatedSubQueryOp
- func (CorrelatedSubQueryOp) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (CorrelatedSubQueryOp) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
- func (c *CorrelatedSubQueryOp) Clone(inputs []ops.Operator) ops.Operator
- func (c *CorrelatedSubQueryOp) IPhysical()
- func (c *CorrelatedSubQueryOp) Inputs() []ops.Operator
- type Cost
- type Delete
- func (Delete) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (Delete) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
- func (d *Delete) Clone(inputs []ops.Operator) ops.Operator
- func (d *Delete) IPhysical()
- func (Delete) Inputs() []ops.Operator
- func (d *Delete) Introduces() semantics.TableSet
- func (d *Delete) TablesUsed() []string
- type Derived
- func (d *Derived) AddColumn(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
- func (d *Derived) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (d *Derived) Clone(inputs []ops.Operator) ops.Operator
- func (d *Derived) IPhysical()
- func (d *Derived) Inputs() []ops.Operator
- func (d *Derived) IsMergeable(ctx *plancontext.PlanningContext) bool
- type Filter
- func (f *Filter) AddColumn(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
- func (f *Filter) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (f *Filter) Clone(inputs []ops.Operator) ops.Operator
- func (f *Filter) Compact(*plancontext.PlanningContext) (ops.Operator, rewrite.TreeIdentity, error)
- func (f *Filter) IPhysical()
- func (f *Filter) Inputs() []ops.Operator
- func (f *Filter) UnsolvedPredicates(st *semantics.SemTable) []sqlparser.Expr
- type GroupBy
- type Horizon
- func (Horizon) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (h *Horizon) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (h *Horizon) Clone(inputs []ops.Operator) ops.Operator
- func (h *Horizon) IPhysical()
- func (h *Horizon) Inputs() []ops.Operator
- type Join
- func (Join) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (j *Join) AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) error
- func (j *Join) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (j *Join) Clone(inputs []ops.Operator) ops.Operator
- func (j *Join) Compact(ctx *plancontext.PlanningContext) (ops.Operator, rewrite.TreeIdentity, error)
- func (j *Join) GetLHS() ops.Operator
- func (j *Join) GetRHS() ops.Operator
- func (j *Join) Inputs() []ops.Operator
- func (j *Join) IsInner() bool
- func (j *Join) MakeInner()
- func (j *Join) SetLHS(operator ops.Operator)
- func (j *Join) SetRHS(operator ops.Operator)
- type JoinOp
- type OrderBy
- type QueryGraph
- func (QueryGraph) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (qg *QueryGraph) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (qg *QueryGraph) Clone(inputs []ops.Operator) ops.Operator
- func (qg *QueryGraph) GetPredicates(lhs, rhs semantics.TableSet) []sqlparser.Expr
- func (QueryGraph) Inputs() []ops.Operator
- func (qg *QueryGraph) Introduces() semantics.TableSet
- func (qg *QueryGraph) UnsolvedPredicates(_ *semantics.SemTable) []sqlparser.Expr
- type QueryProjection
- func (qp *QueryProjection) AddGroupBy(by GroupBy)
- func (qp *QueryProjection) AggrRewriter(ctx *plancontext.PlanningContext) *AggrRewriter
- func (qp *QueryProjection) AggregationExpressions(ctx *plancontext.PlanningContext) (out []Aggr, err error)
- func (qp *QueryProjection) AlignGroupByAndOrderBy(ctx *plancontext.PlanningContext)
- func (qp *QueryProjection) FindSelectExprIndexForExpr(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (*int, *sqlparser.AliasedExpr)
- func (qp *QueryProjection) GetColumnCount() int
- func (qp *QueryProjection) GetGrouping() []GroupBy
- func (qp *QueryProjection) GetSimplifiedExpr(e sqlparser.Expr) (expr sqlparser.Expr, weightStrExpr sqlparser.Expr, err error)
- func (qp *QueryProjection) NeedsAggregation() bool
- func (qp *QueryProjection) NeedsDistinct() bool
- func (qp *QueryProjection) NeedsProjecting(ctx *plancontext.PlanningContext, ...) (needsVtGateEval bool, expressions []sqlparser.Expr, colNames []string, ...)
- type QueryTable
- type Route
- func (r *Route) AddColumn(ctx *plancontext.PlanningContext, e sqlparser.Expr) (int, error)
- func (r *Route) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (r *Route) AlternateInKeyspace(keyspace *vindexes.Keyspace) *Route
- func (r *Route) Clone(inputs []ops.Operator) ops.Operator
- func (r *Route) Cost() int
- func (*Route) IPhysical()
- func (r *Route) Inputs() []ops.Operator
- func (r *Route) IsSingleShard() bool
- func (r *Route) PickBestAvailableVindex()
- func (r *Route) SelectedVindex() vindexes.Vindex
- func (r *Route) TablesUsed() []string
- func (r *Route) UpdateRoutingLogic(ctx *plancontext.PlanningContext, expr sqlparser.Expr) error
- func (r *Route) VindexExpressions() []sqlparser.Expr
- type SelectExpr
- type SubQuery
- type SubQueryInner
- func (SubQueryInner) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (SubQueryInner) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
- func (s *SubQueryInner) Clone(inputs []ops.Operator) ops.Operator
- func (s *SubQueryInner) Inputs() []ops.Operator
- type SubQueryOp
- func (SubQueryOp) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (SubQueryOp) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
- func (s *SubQueryOp) Clone(inputs []ops.Operator) ops.Operator
- func (s *SubQueryOp) IPhysical()
- func (s *SubQueryOp) Inputs() []ops.Operator
- type Table
- func (to *Table) AddCol(col *sqlparser.ColName)
- func (to *Table) AddColumn(_ *plancontext.PlanningContext, e sqlparser.Expr) (int, error)
- func (to *Table) AddPredicate(_ *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (to *Table) Clone([]ops.Operator) ops.Operator
- func (to *Table) GetColumns() []*sqlparser.ColName
- func (to *Table) IPhysical()
- func (Table) Inputs() []ops.Operator
- func (to *Table) Introduces() semantics.TableSet
- func (to *Table) TablesUsed() []string
- type TableIDIntroducer
- type TableUser
- type Union
- func (Union) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (u *Union) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (u *Union) Clone(inputs []ops.Operator) ops.Operator
- func (u *Union) Compact(*plancontext.PlanningContext) (ops.Operator, rewrite.TreeIdentity, error)
- func (u *Union) GetSelectFor(source int) (*sqlparser.Select, error)
- func (u *Union) IPhysical()
- func (u *Union) Inputs() []ops.Operator
- type Update
- func (Update) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
- func (Update) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
- func (u *Update) Clone(inputs []ops.Operator) ops.Operator
- func (u *Update) IPhysical()
- func (Update) Inputs() []ops.Operator
- func (u *Update) Introduces() semantics.TableSet
- func (u *Update) TablesUsed() []string
- type Vindex
- func (v *Vindex) AddCol(col *sqlparser.ColName)
- func (v *Vindex) AddColumn(_ *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
- func (v *Vindex) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
- func (v *Vindex) CheckValid() error
- func (v *Vindex) Clone([]ops.Operator) ops.Operator
- func (v *Vindex) GetColumns() []*sqlparser.ColName
- func (v *Vindex) IPhysical()
- func (Vindex) Inputs() []ops.Operator
- func (v *Vindex) Introduces() semantics.TableSet
- func (v *Vindex) TablesUsed() []string
- type VindexOption
- type VindexPlusPredicates
- type VindexTable
Constants ¶
const VindexUnsupported = "WHERE clause for vindex function must be of the form id = <val> or id in(<val>,...)"
Variables ¶
This section is empty.
Functions ¶
func AddPredicate ¶
func BreakExpressionInLHSandRHS ¶
func BreakExpressionInLHSandRHS( ctx *plancontext.PlanningContext, expr sqlparser.Expr, lhs semantics.TableSet, ) (bvNames []string, columns []*sqlparser.ColName, rewrittenExpr sqlparser.Expr, err error)
BreakExpressionInLHSandRHS takes an expression and extracts the parts that are coming from one of the sides into `ColName`s that are needed
func CheckValid ¶
func Compact ¶
func Compact(ctx *plancontext.PlanningContext, op ops.Operator) (ops.Operator, error)
Compact will optimise the operator tree into a smaller but equivalent version
func CompareRefInt ¶
CompareRefInt compares two references of integers. In case either one is nil, it is considered to be smaller
func PlanQuery ¶
func PlanQuery(ctx *plancontext.PlanningContext, selStmt sqlparser.Statement) (ops.Operator, error)
func QualifiedIdentifier ¶
func QualifiedIdentifier(ks *vindexes.Keyspace, i sqlparser.IdentifierCS) string
func QualifiedTableName ¶
func QualifiedTableNames ¶
func RemovePredicate ¶
func RemovePredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr, op ops.Operator) (ops.Operator, error)
RemovePredicate is used when we turn a predicate into a plan operator, and the predicate needs to be removed as an AST construct
func SingleQualifiedIdentifier ¶
func SingleQualifiedIdentifier(ks *vindexes.Keyspace, i sqlparser.IdentifierCS) []string
func SortAggregations ¶
func SortAggregations(a []Aggr)
func SortGrouping ¶
func SortGrouping(a []GroupBy)
func TablesUsed ¶
func ToSQL ¶
func ToSQL(ctx *plancontext.PlanningContext, op ops.Operator) (sqlparser.SelectStatement, error)
Types ¶
type Aggr ¶
type Aggr struct { Original *sqlparser.AliasedExpr Func sqlparser.AggrFunc OpCode engine.AggregateOpcode Alias string // The index at which the user expects to see this aggregated function. Set to nil, if the user does not ask for it Index *int Distinct bool }
Aggr encodes all information needed for aggregation functions
type AggrRewriter ¶
type AggrRewriter struct { Err error // contains filtered or unexported fields }
func (*AggrRewriter) RewriteDown ¶
RewriteDown stops the walker from entering inside aggregation functions
type ApplyJoin ¶
type ApplyJoin struct {
LHS, RHS ops.Operator
// Columns stores the column indexes of the columns coming from the left and right side
// negative value comes from LHS and positive from RHS
Columns []int
// ColumnsAST keeps track of what AST expression is represented in the Columns array
ColumnsAST []sqlparser.Expr
// Vars are the arguments that need to be copied from the LHS to the RHS
Vars map[string]int
// LeftJoin will be true in the case of an outer join
LeftJoin bool
// JoinCols are the columns from the LHS used for the join.
// These are the same columns pushed on the LHS that are now used in the Vars field
LHSColumns []*sqlparser.ColName
Predicate sqlparser.Expr
}
ApplyJoin is a nested loop join - for each row on the LHS, we'll execute the plan on the RHS, feeding data from left to right
func NewApplyJoin ¶
func (*ApplyJoin) AddColumn ¶
func (a *ApplyJoin) AddColumn(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
func (*ApplyJoin) AddJoinPredicate ¶
func (a *ApplyJoin) AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) error
func (*ApplyJoin) AddPredicate ¶
func (a *ApplyJoin) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
func (*ApplyJoin) IPhysical ¶
func (a *ApplyJoin) IPhysical()
IPhysical implements the PhysicalOperator interface
type ColNameColumns ¶
type CorrelatedSubQueryOp ¶
type CorrelatedSubQueryOp struct {
func (CorrelatedSubQueryOp) AddColumn ¶
func (CorrelatedSubQueryOp) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (CorrelatedSubQueryOp) AddPredicate ¶
func (CorrelatedSubQueryOp) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
AddPredicate implements the Operator interface
func (*CorrelatedSubQueryOp) Clone ¶
func (c *CorrelatedSubQueryOp) Clone(inputs []ops.Operator) ops.Operator
Clone implements the Operator interface
func (*CorrelatedSubQueryOp) IPhysical ¶
func (c *CorrelatedSubQueryOp) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*CorrelatedSubQueryOp) Inputs ¶
func (c *CorrelatedSubQueryOp) Inputs() []ops.Operator
Inputs implements the Operator interface
type Delete ¶
type Delete struct { QTable *QueryTable VTable *vindexes.Table OwnedVindexQuery string AST *sqlparser.Delete // contains filtered or unexported fields }
func (Delete) AddColumn ¶
func (Delete) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (Delete) AddPredicate ¶
func (Delete) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
AddPredicate implements the Operator interface
func (*Delete) IPhysical ¶
func (d *Delete) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*Delete) Introduces ¶
Introduces implements the PhysicalOperator interface
func (*Delete) TablesUsed ¶
type Derived ¶
type Derived struct { Source ops.Operator Query sqlparser.SelectStatement Alias string ColumnAliases sqlparser.Columns // Columns needed to feed other plans Columns []*sqlparser.ColName ColumnsOffset []int }
func (*Derived) AddColumn ¶
func (d *Derived) AddColumn(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
func (*Derived) AddPredicate ¶
func (d *Derived) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
func (*Derived) IPhysical ¶
func (d *Derived) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*Derived) IsMergeable ¶
func (d *Derived) IsMergeable(ctx *plancontext.PlanningContext) bool
IsMergeable is not a great name for this function. Suggestions for a better one are welcome! This function will return false if the derived table inside it has to run on the vtgate side, and so can't be merged with subqueries This logic can also be used to check if this is a derived table that can be had on the left hand side of a vtgate join. Since vtgate joins are always nested loop joins, we can't execute them on the RHS if they do some things, like LIMIT or GROUP BY on wrong columns
type Filter ¶
func (*Filter) AddColumn ¶
func (f *Filter) AddColumn(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
func (*Filter) AddPredicate ¶
func (f *Filter) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
func (*Filter) Compact ¶
func (f *Filter) Compact(*plancontext.PlanningContext) (ops.Operator, rewrite.TreeIdentity, error)
func (*Filter) IPhysical ¶
func (f *Filter) IPhysical()
IPhysical implements the PhysicalOperator interface
type GroupBy ¶
type GroupBy struct { Inner sqlparser.Expr WeightStrExpr sqlparser.Expr // The index at which the user expects to see this column. Set to nil, if the user does not ask for it InnerIndex *int // contains filtered or unexported fields }
GroupBy contains the expression to used in group by and also if grouping is needed at VTGate level then what the weight_string function expression to be sent down for evaluation.
func (GroupBy) AsAliasedExpr ¶
func (b GroupBy) AsAliasedExpr() *sqlparser.AliasedExpr
type Horizon ¶
type Horizon struct { Source ops.Operator Select sqlparser.SelectStatement // contains filtered or unexported fields }
Horizon is an operator we use until we decide how to handle the source to the horizon. It contains information about the planning we have to do after deciding how we will send the query to the tablets.
func (Horizon) AddColumn ¶
func (Horizon) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (*Horizon) AddPredicate ¶
func (h *Horizon) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
type Join ¶
type Join struct {
LHS, RHS ops.Operator
Predicate sqlparser.Expr
LeftJoin bool
// contains filtered or unexported fields
}
Join represents a join. If we have a predicate, this is an inner join. If no predicate exists, it is a cross join
func (Join) AddColumn ¶
func (Join) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (*Join) AddJoinPredicate ¶
func (j *Join) AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) error
func (*Join) AddPredicate ¶
func (j *Join) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
func (*Join) Compact ¶
func (j *Join) Compact(ctx *plancontext.PlanningContext) (ops.Operator, rewrite.TreeIdentity, error)
type OrderBy ¶
OrderBy contains the expression to used in order by and also if ordering is needed at VTGate level then what the weight_string function expression to be sent down for evaluation.
type QueryGraph ¶
type QueryGraph struct { // the Tables, including predicates that only depend on this particular table Tables []*QueryTable // NoDeps contains the predicates that can be evaluated anywhere. NoDeps sqlparser.Expr // contains filtered or unexported fields }
QueryGraph represents the FROM and WHERE parts of a query.
It is an intermediate representation of the query that makes it easier for the planner to find all possible join combinations. Instead of storing the query information in a form that is close to the syntax (AST), we extract the interesting parts into a graph form with the nodes being tables in the FROM clause and the edges between them being predicates. We keep predicates in a hash map keyed by the dependencies of the predicate. This makes it very fast to look up connections between tables in the query.
func (QueryGraph) AddColumn ¶
func (QueryGraph) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (*QueryGraph) AddPredicate ¶
func (qg *QueryGraph) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
func (*QueryGraph) Clone ¶
func (qg *QueryGraph) Clone(inputs []ops.Operator) ops.Operator
Clone implements the Operator interface
func (*QueryGraph) GetPredicates ¶
func (qg *QueryGraph) GetPredicates(lhs, rhs semantics.TableSet) []sqlparser.Expr
GetPredicates returns the predicates that are applicable for the two given TableSets
func (*QueryGraph) Introduces ¶
func (qg *QueryGraph) Introduces() semantics.TableSet
Introduces implements the TableIDIntroducer interface
func (*QueryGraph) UnsolvedPredicates ¶
func (qg *QueryGraph) UnsolvedPredicates(_ *semantics.SemTable) []sqlparser.Expr
UnsolvedPredicates implements the unresolved interface
type QueryProjection ¶
type QueryProjection struct { // If you change the contents here, please update the toString() method SelectExprs []SelectExpr HasAggr bool Distinct bool OrderExprs []OrderBy CanPushDownSorting bool HasStar bool // AddedColumn keeps a counter for expressions added to solve HAVING expressions the user is not selecting AddedColumn int // contains filtered or unexported fields }
QueryProjection contains the information about the projections, group by and order by expressions used to do horizon planning.
func CreateQPFromSelect ¶
func CreateQPFromSelect(ctx *plancontext.PlanningContext, sel *sqlparser.Select) (*QueryProjection, error)
CreateQPFromSelect creates the QueryProjection for the input *sqlparser.Select
func CreateQPFromUnion ¶
func CreateQPFromUnion(union *sqlparser.Union) (*QueryProjection, error)
CreateQPFromUnion creates the QueryProjection for the input *sqlparser.Union
func (*QueryProjection) AddGroupBy ¶
func (qp *QueryProjection) AddGroupBy(by GroupBy)
AddGroupBy does just that
func (*QueryProjection) AggrRewriter ¶
func (qp *QueryProjection) AggrRewriter(ctx *plancontext.PlanningContext) *AggrRewriter
AggrRewriter extracts
func (*QueryProjection) AggregationExpressions ¶
func (qp *QueryProjection) AggregationExpressions(ctx *plancontext.PlanningContext) (out []Aggr, err error)
func (*QueryProjection) AlignGroupByAndOrderBy ¶
func (qp *QueryProjection) AlignGroupByAndOrderBy(ctx *plancontext.PlanningContext)
AlignGroupByAndOrderBy aligns the group by and order by columns, so they are in the same order The GROUP BY clause is a set - the order between the elements does not make any difference, so we can simply re-arrange the column order We are also free to add more ORDER BY columns than the user asked for which we leverage, so the input is already ordered according to the GROUP BY columns used
func (*QueryProjection) FindSelectExprIndexForExpr ¶
func (qp *QueryProjection) FindSelectExprIndexForExpr(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (*int, *sqlparser.AliasedExpr)
FindSelectExprIndexForExpr returns the index of the given expression in the select expressions, if it is part of it returns -1 otherwise.
func (*QueryProjection) GetColumnCount ¶
func (qp *QueryProjection) GetColumnCount() int
func (*QueryProjection) GetGrouping ¶
func (qp *QueryProjection) GetGrouping() []GroupBy
GetGrouping returns a copy of the grouping parameters of the QP
func (*QueryProjection) GetSimplifiedExpr ¶
func (qp *QueryProjection) GetSimplifiedExpr(e sqlparser.Expr) (expr sqlparser.Expr, weightStrExpr sqlparser.Expr, err error)
GetSimplifiedExpr takes an expression used in ORDER BY or GROUP BY, and returns an expression that is simpler to evaluate
func (*QueryProjection) NeedsAggregation ¶
func (qp *QueryProjection) NeedsAggregation() bool
NeedsAggregation returns true if we either have aggregate functions or grouping defined
func (*QueryProjection) NeedsDistinct ¶
func (qp *QueryProjection) NeedsDistinct() bool
NeedsDistinct returns true if the query needs explicit distinct
func (*QueryProjection) NeedsProjecting ¶ added in v0.16.1
func (qp *QueryProjection) NeedsProjecting( ctx *plancontext.PlanningContext, pusher func(expr *sqlparser.AliasedExpr) (int, error), ) (needsVtGateEval bool, expressions []sqlparser.Expr, colNames []string, err error)
NeedsProjecting returns true if we have projections that need to be evaluated at the vtgate level and can't be pushed down to MySQL
type QueryTable ¶
type QueryTable struct { ID semantics.TableSet Alias *sqlparser.AliasedTableExpr Table sqlparser.TableName Predicates []sqlparser.Expr IsInfSchema bool }
QueryTable is a single FROM table, including all predicates particular to this table This is to be used as an immutable data structure which is created in the logical Operator Tree
func (*QueryTable) Clone ¶
func (qt *QueryTable) Clone() *QueryTable
Clone implements the Operator interface
type Route ¶
type Route struct { Source ops.Operator RouteOpCode engine.Opcode Keyspace *vindexes.Keyspace // here we store the possible vindexes we can use so that when we add predicates to the plan, // we can quickly check if the new predicates enables any new vindex Options VindexPreds []*VindexPlusPredicates // the best option available is stored here Selected *VindexOption // The following two fields are used when routing information_schema queries SysTableTableSchema []evalengine.Expr SysTableTableName map[string]evalengine.Expr // SeenPredicates contains all the predicates that have had a chance to influence routing. // If we need to replan routing, we'll use this list SeenPredicates []sqlparser.Expr // TargetDestination specifies an explicit target destination tablet type TargetDestination key.Destination // Alternates contains alternate routes to equivalent sources in // other keyspaces. Alternates map[*vindexes.Keyspace]*Route // Routes that have been merged into this one. MergedWith []*Route }
func (*Route) AddColumn ¶
func (r *Route) AddColumn(ctx *plancontext.PlanningContext, e sqlparser.Expr) (int, error)
func (*Route) AddPredicate ¶
func (r *Route) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
func (*Route) AlternateInKeyspace ¶
func (*Route) IPhysical ¶
func (*Route) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*Route) IsSingleShard ¶
func (*Route) PickBestAvailableVindex ¶
func (r *Route) PickBestAvailableVindex()
PickBestAvailableVindex goes over the available vindexes for this route and picks the best one available.
func (*Route) SelectedVindex ¶
func (*Route) TablesUsed ¶
TablesUsed returns tables used by MergedWith routes, which are not included in Inputs() and thus not a part of the operator tree
func (*Route) UpdateRoutingLogic ¶
func (r *Route) UpdateRoutingLogic(ctx *plancontext.PlanningContext, expr sqlparser.Expr) error
func (*Route) VindexExpressions ¶
type SelectExpr ¶
type SelectExpr struct { Col sqlparser.SelectExpr Aggr bool }
SelectExpr provides whether the columns is aggregation expression or not.
func (SelectExpr) GetAliasedExpr ¶
func (s SelectExpr) GetAliasedExpr() (*sqlparser.AliasedExpr, error)
GetAliasedExpr returns the SelectExpr as a *sqlparser.AliasedExpr if its type allows it, otherwise an error is returned.
type SubQuery ¶
type SubQuery struct { Outer ops.Operator Inner []*SubQueryInner // contains filtered or unexported fields }
SubQuery stores the information about subquery
func (SubQuery) AddColumn ¶
func (SubQuery) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (SubQuery) AddPredicate ¶
func (SubQuery) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
AddPredicate implements the Operator interface
type SubQueryInner ¶
type SubQueryInner struct { // Inner is the Operator inside the parenthesis of the subquery. // i.e: select (select 1 union select 1), the Inner here would be // of type Concatenate since we have a Union. Inner ops.Operator // ExtractedSubquery contains all information we need about this subquery ExtractedSubquery *sqlparser.ExtractedSubquery // contains filtered or unexported fields }
SubQueryInner stores the subquery information for a select statement
func (SubQueryInner) AddColumn ¶
func (SubQueryInner) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (SubQueryInner) AddPredicate ¶
func (SubQueryInner) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
AddPredicate implements the Operator interface
func (*SubQueryInner) Clone ¶
func (s *SubQueryInner) Clone(inputs []ops.Operator) ops.Operator
Clone implements the Operator interface
func (*SubQueryInner) Inputs ¶
func (s *SubQueryInner) Inputs() []ops.Operator
Inputs implements the Operator interface
type SubQueryOp ¶
type SubQueryOp struct {
Outer, Inner ops.Operator
Extracted *sqlparser.ExtractedSubquery
// contains filtered or unexported fields
}
func (SubQueryOp) AddColumn ¶
func (SubQueryOp) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (SubQueryOp) AddPredicate ¶
func (SubQueryOp) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
AddPredicate implements the Operator interface
func (*SubQueryOp) Clone ¶
func (s *SubQueryOp) Clone(inputs []ops.Operator) ops.Operator
Clone implements the Operator interface
func (*SubQueryOp) IPhysical ¶
func (s *SubQueryOp) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*SubQueryOp) Inputs ¶
func (s *SubQueryOp) Inputs() []ops.Operator
Inputs implements the Operator interface
type Table ¶
type Table struct { QTable *QueryTable VTable *vindexes.Table Columns []*sqlparser.ColName // contains filtered or unexported fields }
func (*Table) AddColumn ¶
func (to *Table) AddColumn(_ *plancontext.PlanningContext, e sqlparser.Expr) (int, error)
func (*Table) AddPredicate ¶
func (to *Table) AddPredicate(_ *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
AddPredicate implements the PhysicalOperator interface
func (*Table) GetColumns ¶
func (*Table) IPhysical ¶
func (to *Table) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*Table) Introduces ¶
Introduces implements the PhysicalOperator interface
func (*Table) TablesUsed ¶
type TableIDIntroducer ¶
TableIDIntroducer is used to signal that this operator introduces data from a new source
type TableUser ¶
type TableUser interface {
TablesUsed() []string
}
TableUser is used to signal that this operator directly interacts with one or more tables
type Union ¶
type Union struct { Sources []ops.Operator Distinct bool // TODO this should be removed. For now it's used to fail queries Ordering sqlparser.OrderBy // contains filtered or unexported fields }
func (Union) AddColumn ¶
func (Union) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (*Union) AddPredicate ¶
func (u *Union) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
AddPredicate adds a predicate a UNION by pushing the predicate to all sources of the UNION.
this is done by offset and expression rewriting. Say we have a query like so:
select * (
select foo as col, bar from tbl1 union select id, baz from tbl2
) as X where X.col = 42
We want to push down the `X.col = 42` as far down the operator tree as possible. We want to end up with an operator tree that looks something like this:
select * (
select foo as col, bar from tbl1 where foo = 42 union select id, baz from tbl2 where id = 42
) as X
Notice how `X.col = 42` has been translated to `foo = 42` and `id = 42` on respective WHERE clause. The first SELECT of the union dictates the column names, and the second is whatever expression can be found on the same offset. The names of the RHS are discarded.
func (*Union) Compact ¶
func (u *Union) Compact(*plancontext.PlanningContext) (ops.Operator, rewrite.TreeIdentity, error)
type Update ¶
type Update struct { QTable *QueryTable VTable *vindexes.Table Assignments map[string]sqlparser.Expr ChangedVindexValues map[string]*engine.VindexValues OwnedVindexQuery string AST *sqlparser.Update // contains filtered or unexported fields }
func (Update) AddColumn ¶
func (Update) AddColumn(*plancontext.PlanningContext, sqlparser.Expr) (int, error)
AddColumn implements the Operator interface
func (Update) AddPredicate ¶
func (Update) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) (ops.Operator, error)
AddPredicate implements the Operator interface
func (*Update) IPhysical ¶
func (u *Update) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*Update) Introduces ¶
Introduces implements the PhysicalOperator interface
func (*Update) TablesUsed ¶
type Vindex ¶
type Vindex struct { OpCode engine.VindexOpcode Table VindexTable Vindex vindexes.Vindex Solved semantics.TableSet Columns []*sqlparser.ColName Value sqlparser.Expr // contains filtered or unexported fields }
func (*Vindex) AddColumn ¶
func (v *Vindex) AddColumn(_ *plancontext.PlanningContext, expr sqlparser.Expr) (int, error)
func (*Vindex) AddPredicate ¶
func (v *Vindex) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) (ops.Operator, error)
func (*Vindex) CheckValid ¶
checkValid implements the Operator interface
func (*Vindex) GetColumns ¶
func (*Vindex) IPhysical ¶
func (v *Vindex) IPhysical()
IPhysical implements the PhysicalOperator interface
func (*Vindex) Introduces ¶
Introduces implements the Operator interface
func (*Vindex) TablesUsed ¶
TablesUsed implements the Operator interface. It is not keyspace-qualified.
type VindexOption ¶
type VindexOption struct { Ready bool Values []evalengine.Expr // columns that we have seen so far. Used only for multi-column vindexes so that we can track how many columns part of the vindex we have seen ColsSeen map[string]any ValueExprs []sqlparser.Expr Predicates []sqlparser.Expr OpCode engine.Opcode FoundVindex vindexes.Vindex Cost Cost }
VindexOption stores the information needed to know if we have all the information needed to use a vindex
type VindexPlusPredicates ¶
type VindexPlusPredicates struct { TableID semantics.TableSet ColVindex *vindexes.ColumnVindex // during planning, we store the alternatives found for this route in this slice Options []*VindexOption }
VindexPlusPredicates is a struct used to store all the predicates that the vindex can be used to query
Source Files ¶
- SQL_builder.go
- apply_join.go
- correlated_subquery.go
- delete.go
- derived.go
- dml_planning.go
- expressions.go
- filter.go
- helpers.go
- horizon.go
- horizon_planning.go
- join.go
- joins.go
- logical.go
- operator.go
- operator_funcs.go
- querygraph.go
- queryprojection.go
- route.go
- route_planning.go
- subquery.go
- subquery_planning.go
- system_tables.go
- table.go
- union.go
- update.go
- vindex.go