operators

package
v0.19.4 Latest Latest
Warning

This package is not in the latest version of its module.

Go to latest
Published: May 8, 2024 License: Apache-2.0 Imports: 33 Imported by: 0

Documentation

Overview

Package operators contains the operators used to plan queries.

The operators go through a few phases while planning:

  1. Initial plan In this first pass, we build an operator tree from the incoming parsed query. At the leaves, it will contain QueryGraphs - these are the tables in the FROM clause that we can easily do join ordering on because they are all inner joins. All the post-processing - aggregations, sorting, limit etc. are at this stage contained in Horizon structs. We try to push these down under routes, and expand the ones that can't be pushed down into individual operators such as Projection, Aggregation, Limit, etc.
  2. Planning Once the initial plan has been fully built, we go through a number of phases. recursively running rewriters on the tree in a fixed point fashion, until we've gone over all phases and the tree has stop changing.
  3. Offset planning Now is the time to stop working with AST objects and transform remaining expressions being used on top of vtgate to either offsets on inputs or evalengine expressions.

Package operators contains the operators used to plan queries.

The operators go through a few phases while planning:

  1. Initial plan In this first pass, we build an operator tree from the incoming parsed query. At the leaves, it will contain QueryGraphs - these are the tables in the FROM clause that we can easily do join ordering on because they are all inner joins. All the post-processing - aggregations, sorting, limit etc. are at this stage contained in Horizon structs. We try to push these down under routes, and expand the ones that can't be pushed down into individual operators such as Projection, Agreggation, Limit, etc.
  2. Planning Once the initial plan has been fully built, we go through a number of phases. recursively running rewriters on the tree in a fixed point fashion, until we've gone over all phases and the tree has stop changing.
  3. Offset planning Now is the time to stop working with AST objects and transform remaining expressions being used on top of vtgate to either offsets on inputs or evalengine expressions.

Index

Constants

View Source
const (
	Unknown joinSide = iota
	Left
	Right
)

Variables

View Source
var DebugOperatorTree = false

Functions

func CompareRefInt

func CompareRefInt(a *int, b *int) bool

CompareRefInt compares two references of integers. In case either one is nil, it is considered to be smaller

func CostOf

func CostOf(op Operator) (cost int)

func EnableDebugPrinting added in v0.19.0

func EnableDebugPrinting() (reset func())

func IsTableSchemaOrName added in v0.17.0

func IsTableSchemaOrName(e sqlparser.Expr, version string) (col *sqlparser.ColName, isTableSchema bool, isTableName bool)

func NotImplementedSchemaInfoResolver added in v0.17.0

func NotImplementedSchemaInfoResolver(*sqlparser.ColName) (int, error)

func PanicHandler added in v0.19.0

func PanicHandler(err *error)

func QualifiedIdentifier

func QualifiedIdentifier(ks *vindexes.Keyspace, i sqlparser.IdentifierCS) string

func QualifiedString

func QualifiedString(ks *vindexes.Keyspace, s string) string

func QualifiedTableName

func QualifiedTableName(ks *vindexes.Keyspace, t sqlparser.TableName) string

func QualifiedTableNames

func QualifiedTableNames(ks *vindexes.Keyspace, ts []sqlparser.TableName) []string

func QualifiedTables

func QualifiedTables(ks *vindexes.Keyspace, vts []*vindexes.Table) []string

func SingleQualifiedIdentifier

func SingleQualifiedIdentifier(ks *vindexes.Keyspace, i sqlparser.IdentifierCS) []string

func SingleQualifiedString

func SingleQualifiedString(ks *vindexes.Keyspace, s string) []string

func SortGrouping

func SortGrouping(a []GroupBy)

func Swap added in v0.19.0

func Swap(parent, child Operator, message string) (Operator, *ApplyResult)

Swap takes a tree like a->b->c and swaps `a` and `b`, so we end up with b->a->c

func TableID

func TableID(op Operator) (result semantics.TableSet)

func TablesUsed

func TablesUsed(op Operator) []string

func ToTree added in v0.19.0

func ToTree(op Operator) string

ToTree returns the operator as ascii tree. Should only be used for debugging

func Visit added in v0.19.0

func Visit(root Operator, visitor func(Operator) error) error

Visit allows for the walking of the operator tree. If any error is returned, the walk is aborted

Types

type Aggr

type Aggr struct {
	Original *sqlparser.AliasedExpr
	Func     sqlparser.AggrFunc
	OpCode   opcode.AggregateOpcode

	// OriginalOpCode will contain opcode.AggregateUnassigned unless we are changing opcode while pushing them down
	OriginalOpCode opcode.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
	// Only used in the old Horizon Planner
	Index *int

	Distinct bool

	// the offsets point to columns on the same aggregator
	ColOffset int
	WSOffset  int

	SubQueryExpression []*SubQuery
}

Aggr encodes all information needed for aggregation functions

func NewAggr added in v0.17.0

func NewAggr(opCode opcode.AggregateOpcode, f sqlparser.AggrFunc, original *sqlparser.AliasedExpr, alias string) Aggr

func (Aggr) GetTypeCollation added in v0.17.2

func (aggr Aggr) GetTypeCollation(ctx *plancontext.PlanningContext) evalengine.Type

func (Aggr) NeedsWeightString added in v0.18.0

func (aggr Aggr) NeedsWeightString(ctx *plancontext.PlanningContext) bool

type AggrRewriter

type AggrRewriter struct {
	// contains filtered or unexported fields
}

func (*AggrRewriter) RewriteDown

func (ar *AggrRewriter) RewriteDown() func(sqlparser.SQLNode, sqlparser.SQLNode) bool

RewriteDown stops the walker from entering inside aggregation functions

func (*AggrRewriter) RewriteUp

func (ar *AggrRewriter) RewriteUp() func(*sqlparser.Cursor) bool

RewriteUp will go through an expression, add aggregations to the QP, and rewrite them to use column offset

type Aggregator added in v0.17.0

type Aggregator struct {
	Source  Operator
	Columns []*sqlparser.AliasedExpr

	Grouping     []GroupBy
	Aggregations []Aggr

	// We support a single distinct aggregation per aggregator. It is stored here.
	// When planning the ordering that the OrderedAggregate will require,
	// this needs to be the last ORDER BY expression
	DistinctExpr sqlparser.Expr

	// Pushed will be set to true once this aggregation has been pushed deeper in the tree
	Pushed bool

	// Original will only be true for the original aggregator created from the AST
	Original      bool
	ResultColumns int

	QP *QueryProjection

	DT *DerivedTable
	// contains filtered or unexported fields
}

Aggregator represents a GroupBy γ relational operator. Both all aggregations and no grouping, and the inverse of all grouping and no aggregations are valid configurations of this operator

func (*Aggregator) AddColumn added in v0.17.0

func (a *Aggregator) AddColumn(ctx *plancontext.PlanningContext, reuse bool, groupBy bool, ae *sqlparser.AliasedExpr) int

func (*Aggregator) AddPredicate added in v0.17.0

func (a *Aggregator) AddPredicate(_ *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Aggregator) Clone added in v0.17.0

func (a *Aggregator) Clone(inputs []Operator) Operator

func (*Aggregator) FindCol added in v0.18.0

func (a *Aggregator) FindCol(ctx *plancontext.PlanningContext, in sqlparser.Expr, underRoute bool) int

func (*Aggregator) GetColumns added in v0.17.0

func (*Aggregator) GetOrdering added in v0.17.0

func (a *Aggregator) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*Aggregator) GetSelectExprs added in v0.18.0

func (*Aggregator) Inputs added in v0.17.0

func (a *Aggregator) Inputs() []Operator

func (*Aggregator) SetInputs added in v0.17.0

func (a *Aggregator) SetInputs(operators []Operator)

func (*Aggregator) ShortDescription added in v0.17.0

func (a *Aggregator) ShortDescription() string

func (*Aggregator) SplitAggregatorBelowRoute added in v0.18.0

func (a *Aggregator) SplitAggregatorBelowRoute(input []Operator) *Aggregator

SplitAggregatorBelowRoute returns the aggregator that will live under the Route. This is used when we are splitting the aggregation so one part is done at the mysql level and one part at the vtgate level

type AliasedProjections added in v0.18.0

type AliasedProjections []*ProjExpr

Used when we know all the columns

func (AliasedProjections) GetColumns added in v0.18.0

func (ap AliasedProjections) GetColumns() []*sqlparser.AliasedExpr

func (AliasedProjections) GetSelectExprs added in v0.18.0

func (ap AliasedProjections) GetSelectExprs() sqlparser.SelectExprs

type AnyShardRouting added in v0.17.0

type AnyShardRouting struct {
	Alternates map[*vindexes.Keyspace]*Route
	// contains filtered or unexported fields
}

AnyShardRouting is used for routing logic where any shard in the keyspace can be used. Shared by unsharded and reference routing

func (*AnyShardRouting) AlternateInKeyspace added in v0.17.0

func (rr *AnyShardRouting) AlternateInKeyspace(keyspace *vindexes.Keyspace) *Route

func (*AnyShardRouting) Clone added in v0.17.0

func (rr *AnyShardRouting) Clone() Routing

func (*AnyShardRouting) Cost added in v0.17.0

func (rr *AnyShardRouting) Cost() int

func (*AnyShardRouting) Keyspace added in v0.17.0

func (rr *AnyShardRouting) Keyspace() *vindexes.Keyspace

func (*AnyShardRouting) OpCode added in v0.17.0

func (rr *AnyShardRouting) OpCode() engine.Opcode

func (*AnyShardRouting) UpdateRoutingParams added in v0.17.0

func (rr *AnyShardRouting) UpdateRoutingParams(_ *plancontext.PlanningContext, rp *engine.RoutingParameters)

type ApplyJoin

type ApplyJoin struct {
	LHS, RHS Operator

	// LeftJoin will be true in the case of an outer join
	LeftJoin bool

	// JoinColumns keeps track of what AST expression is represented in the Columns array
	JoinColumns *applyJoinColumns

	// JoinPredicates are join predicates that have been broken up into left hand side and right hand side parts.
	JoinPredicates *applyJoinColumns

	// ExtraVars are columns we need to copy from left to right not needed by any predicates or projections,
	// these are needed by other operators further down the right hand side of the join
	ExtraLHSVars []BindVarExpr

	// 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

	// Vars are the arguments that need to be copied from the LHS to the RHS
	Vars map[string]int
}

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 NewApplyJoin(ctx *plancontext.PlanningContext, lhs, rhs Operator, predicate sqlparser.Expr, leftOuterJoin bool) *ApplyJoin

func (*ApplyJoin) AddColumn

func (aj *ApplyJoin) AddColumn(
	ctx *plancontext.PlanningContext,
	reuse bool,
	groupBy bool,
	expr *sqlparser.AliasedExpr,
) int

func (*ApplyJoin) AddJoinPredicate

func (aj *ApplyJoin) AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr)

func (*ApplyJoin) AddPredicate

func (aj *ApplyJoin) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*ApplyJoin) Clone

func (aj *ApplyJoin) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*ApplyJoin) FindCol added in v0.18.0

func (aj *ApplyJoin) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, _ bool) int

func (*ApplyJoin) GetColumns added in v0.17.0

func (*ApplyJoin) GetLHS

func (aj *ApplyJoin) GetLHS() Operator

func (*ApplyJoin) GetOrdering added in v0.17.0

func (aj *ApplyJoin) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*ApplyJoin) GetRHS

func (aj *ApplyJoin) GetRHS() Operator

func (*ApplyJoin) GetSelectExprs added in v0.18.0

func (aj *ApplyJoin) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*ApplyJoin) Inputs

func (aj *ApplyJoin) Inputs() []Operator

Inputs implements the Operator interface

func (*ApplyJoin) IsInner

func (aj *ApplyJoin) IsInner() bool

func (*ApplyJoin) LHSColumnsNeeded added in v0.18.0

func (a *ApplyJoin) LHSColumnsNeeded(ctx *plancontext.PlanningContext) (needed sqlparser.Exprs)

func (*ApplyJoin) MakeInner

func (aj *ApplyJoin) MakeInner()

func (*ApplyJoin) SetInputs added in v0.17.0

func (aj *ApplyJoin) SetInputs(inputs []Operator)

SetInputs implements the Operator interface

func (*ApplyJoin) SetLHS

func (aj *ApplyJoin) SetLHS(operator Operator)

func (*ApplyJoin) SetRHS

func (aj *ApplyJoin) SetRHS(operator Operator)

func (*ApplyJoin) ShortDescription added in v0.17.0

func (aj *ApplyJoin) ShortDescription() string

type ApplyResult added in v0.19.0

type ApplyResult struct {
	Transformations []Rewrite
}

ApplyResult tracks modifications to node and expression trees. Only return SameTree when it is acceptable to return the original input and discard the returned result as a performance improvement.

var (
	NoRewrite *ApplyResult = nil
)

func Rewrote added in v0.19.0

func Rewrote(message string) *ApplyResult

func (*ApplyResult) Changed added in v0.19.0

func (ar *ApplyResult) Changed() bool

func (*ApplyResult) Merge added in v0.19.0

func (ar *ApplyResult) Merge(other *ApplyResult) *ApplyResult

type BindVarExpr added in v0.18.0

type BindVarExpr struct {
	Name string
	Expr sqlparser.Expr
}

BindVarExpr is an expression needed from one side of a join/subquery, and the argument name for it. TODO: Do we really need to store the name here? it could be found in the semantic state instead

func (BindVarExpr) String added in v0.18.0

func (bve BindVarExpr) String() string

type ColNameColumns

type ColNameColumns interface {
	GetColNames() []*sqlparser.ColName
	AddCol(*sqlparser.ColName)
}

type Comparison added in v0.19.0

type Comparison struct {
	LHS, RHS sqlparser.Expr
}

func (Comparison) String added in v0.19.0

func (c Comparison) String() string

type Cost

type Cost struct {
	VindexCost int
	IsUnique   bool
	OpCode     engine.Opcode
}

Cost is used to make it easy to compare the Cost of two plans with each other

type Delete

type Delete struct {
	Target           TargetTable
	OwnedVindexQuery *sqlparser.Select
	Ignore           bool
	Source           Operator
	// contains filtered or unexported fields
}

func (Delete) AddColumn

func (Delete) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (Delete) AddPredicate

func (Delete) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*Delete) Clone

func (d *Delete) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (Delete) FindCol added in v0.18.0

func (Delete) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (Delete) GetColumns added in v0.17.0

func (Delete) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Delete) GetOrdering added in v0.17.0

func (d *Delete) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (Delete) GetSelectExprs added in v0.18.0

func (Delete) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*Delete) Inputs

func (d *Delete) Inputs() []Operator

func (*Delete) SetInputs added in v0.17.0

func (d *Delete) SetInputs(inputs []Operator)

func (*Delete) ShortDescription added in v0.17.0

func (d *Delete) ShortDescription() string

func (*Delete) TablesUsed

func (d *Delete) TablesUsed() []string

type DeleteWithInput added in v0.19.0

type DeleteWithInput struct {
	Source Operator
	Delete Operator

	Offsets []int
	// contains filtered or unexported fields
}

func (DeleteWithInput) AddColumn added in v0.19.0

func (DeleteWithInput) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (DeleteWithInput) AddPredicate added in v0.19.0

func (DeleteWithInput) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*DeleteWithInput) Clone added in v0.19.0

func (d *DeleteWithInput) Clone(inputs []Operator) Operator

func (DeleteWithInput) FindCol added in v0.19.0

func (DeleteWithInput) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (DeleteWithInput) GetColumns added in v0.19.0

func (DeleteWithInput) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*DeleteWithInput) GetOrdering added in v0.19.0

func (d *DeleteWithInput) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (DeleteWithInput) GetSelectExprs added in v0.19.0

func (DeleteWithInput) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*DeleteWithInput) Inputs added in v0.19.0

func (d *DeleteWithInput) Inputs() []Operator

func (*DeleteWithInput) SetInputs added in v0.19.0

func (d *DeleteWithInput) SetInputs(inputs []Operator)

func (*DeleteWithInput) ShortDescription added in v0.19.0

func (d *DeleteWithInput) ShortDescription() string

type DerivedTable added in v0.18.0

type DerivedTable struct {
	TableID semantics.TableSet
	Alias   string
	Columns sqlparser.Columns
}

func (*DerivedTable) RewriteExpression added in v0.18.0

func (dt *DerivedTable) RewriteExpression(ctx *plancontext.PlanningContext, expr sqlparser.Expr) sqlparser.Expr

func (*DerivedTable) String added in v0.18.0

func (dt *DerivedTable) String() string

type Distinct added in v0.17.0

type Distinct struct {
	Source Operator
	QP     *QueryProjection

	// When we go from AST to operator, we place DISTINCT ops in the required places in the op tree
	// These are marked as `Required`, because they are semantically important to the results of the query.
	// During planning, when we can't push down the DISTINCT op any further, we sometimes create and push down
	// additional DISTINCT ops that are not strictly required, but that limit the number of incoming rows so less
	// work has to be done. When we have pushed down these performance DISTINCTs, we set the `PushedPerformance`
	// field to true on the originating op
	Required          bool
	PushedPerformance bool

	// This is only filled in during offset planning
	Columns []engine.CheckCol

	Truncate int
}

func (*Distinct) AddColumn added in v0.17.0

func (d *Distinct) AddColumn(ctx *plancontext.PlanningContext, reuse bool, gb bool, expr *sqlparser.AliasedExpr) int

func (*Distinct) AddPredicate added in v0.17.0

func (d *Distinct) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Distinct) Clone added in v0.17.0

func (d *Distinct) Clone(inputs []Operator) Operator

func (*Distinct) FindCol added in v0.18.0

func (d *Distinct) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Distinct) GetColumns added in v0.17.0

func (d *Distinct) GetColumns(ctx *plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Distinct) GetOrdering added in v0.17.0

func (d *Distinct) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*Distinct) GetSelectExprs added in v0.18.0

func (d *Distinct) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*Distinct) Inputs added in v0.17.0

func (d *Distinct) Inputs() []Operator

func (*Distinct) SetInputs added in v0.17.0

func (d *Distinct) SetInputs(operators []Operator)

func (*Distinct) ShortDescription added in v0.17.0

func (d *Distinct) ShortDescription() string

type DualRouting added in v0.17.0

type DualRouting struct{}

DualRouting represents the dual-table. It is special compared to all other tables because it can be merged with tables in any keyspace

func (*DualRouting) Clone added in v0.17.0

func (dr *DualRouting) Clone() Routing

func (*DualRouting) Cost added in v0.17.0

func (dr *DualRouting) Cost() int

func (*DualRouting) Keyspace added in v0.17.0

func (dr *DualRouting) Keyspace() *vindexes.Keyspace

func (*DualRouting) OpCode added in v0.17.0

func (dr *DualRouting) OpCode() engine.Opcode

func (*DualRouting) UpdateRoutingParams added in v0.17.0

func (dr *DualRouting) UpdateRoutingParams(*plancontext.PlanningContext, *engine.RoutingParameters)

type EvalEngine added in v0.18.0

type EvalEngine struct {
	EExpr evalengine.Expr
}

EvalEngine is used for expressions that have to be evaluated in the vtgate using the evalengine

type ExprInfo added in v0.18.0

type ExprInfo interface {
	// contains filtered or unexported methods
}

type Filter

type Filter struct {
	Source     Operator
	Predicates []sqlparser.Expr

	// PredicateWithOffsets is the evalengine expression that will finally be used.
	// It contains the ANDed predicates in Predicates, with ColName:s replaced by Offset:s
	PredicateWithOffsets evalengine.Expr

	Truncate int
}

func (*Filter) AddColumn

func (f *Filter) AddColumn(ctx *plancontext.PlanningContext, reuse bool, gb bool, expr *sqlparser.AliasedExpr) int

func (*Filter) AddPredicate

func (f *Filter) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Filter) Clone

func (f *Filter) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*Filter) Compact

func (*Filter) FindCol added in v0.18.0

func (f *Filter) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Filter) GetColumns added in v0.17.0

func (f *Filter) GetColumns(ctx *plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Filter) GetOrdering added in v0.17.0

func (f *Filter) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*Filter) GetSelectExprs added in v0.18.0

func (f *Filter) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*Filter) Inputs

func (f *Filter) Inputs() []Operator

Inputs implements the Operator interface

func (*Filter) SetInputs added in v0.17.0

func (f *Filter) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Filter) ShortDescription added in v0.17.0

func (f *Filter) ShortDescription() string

func (*Filter) UnsolvedPredicates

func (f *Filter) UnsolvedPredicates(st *semantics.SemTable) []sqlparser.Expr

UnsolvedPredicates implements the unresolved interface

type FkCascade added in v0.18.0

type FkCascade struct {
	Selection Operator
	Children  []*FkChild
	Parent    Operator
	// contains filtered or unexported fields
}

FkCascade is used to represent a foreign key cascade operation as an operator. This operator is created for DML queries that require cascades (for example, ON DELETE CASCADE).

func (FkCascade) AddColumn added in v0.18.0

func (FkCascade) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (FkCascade) AddPredicate added in v0.18.0

func (FkCascade) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*FkCascade) Clone added in v0.18.0

func (fkc *FkCascade) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (FkCascade) FindCol added in v0.18.0

func (FkCascade) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (FkCascade) GetColumns added in v0.18.0

func (FkCascade) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*FkCascade) GetOrdering added in v0.18.0

func (fkc *FkCascade) GetOrdering(*plancontext.PlanningContext) []OrderBy

GetOrdering implements the Operator interface

func (FkCascade) GetSelectExprs added in v0.18.0

func (FkCascade) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*FkCascade) Inputs added in v0.18.0

func (fkc *FkCascade) Inputs() []Operator

Inputs implements the Operator interface

func (*FkCascade) SetInputs added in v0.18.0

func (fkc *FkCascade) SetInputs(operators []Operator)

SetInputs implements the Operator interface

func (*FkCascade) ShortDescription added in v0.18.0

func (fkc *FkCascade) ShortDescription() string

ShortDescription implements the Operator interface

type FkChild added in v0.18.0

type FkChild struct {
	BVName         string
	Cols           []int // indexes
	NonLiteralInfo []engine.NonLiteralUpdateInfo
	Op             Operator
	// contains filtered or unexported fields
}

FkChild is used to represent a foreign key child table operation

func (FkChild) AddColumn added in v0.18.0

func (FkChild) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (FkChild) AddPredicate added in v0.18.0

func (FkChild) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (FkChild) FindCol added in v0.18.0

func (FkChild) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (FkChild) GetColumns added in v0.18.0

func (FkChild) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (FkChild) GetSelectExprs added in v0.18.0

func (FkChild) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

type FkVerify added in v0.18.0

type FkVerify struct {
	Verify []*VerifyOp
	Input  Operator
	// contains filtered or unexported fields
}

FkVerify is used to represent a foreign key verification operation as an operator. This operator is created for DML queries that require verifications on the existence of the rows in the parent table (for example, INSERT and UPDATE).

func (FkVerify) AddColumn added in v0.18.0

func (FkVerify) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (FkVerify) AddPredicate added in v0.18.0

func (FkVerify) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*FkVerify) Clone added in v0.18.0

func (fkv *FkVerify) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (FkVerify) FindCol added in v0.18.0

func (FkVerify) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (FkVerify) GetColumns added in v0.18.0

func (FkVerify) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*FkVerify) GetOrdering added in v0.18.0

func (fkv *FkVerify) GetOrdering(*plancontext.PlanningContext) []OrderBy

GetOrdering implements the Operator interface

func (FkVerify) GetSelectExprs added in v0.18.0

func (FkVerify) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*FkVerify) Inputs added in v0.18.0

func (fkv *FkVerify) Inputs() []Operator

Inputs implements the Operator interface

func (*FkVerify) SetInputs added in v0.18.0

func (fkv *FkVerify) SetInputs(operators []Operator)

SetInputs implements the Operator interface

func (*FkVerify) ShortDescription added in v0.18.0

func (fkv *FkVerify) ShortDescription() string

ShortDescription implements the Operator interface

type Generate added in v0.17.0

type Generate struct {
	// Keyspace represents the keyspace information for the table.
	Keyspace *vindexes.Keyspace
	// TableName represents the name of the table.
	TableName sqlparser.TableName

	// Values are the supplied values for the column, which
	// will be stored as a list within the expression. New
	// values will be generated based on how many were not
	// supplied (NULL).
	Values evalengine.Expr
	// Insert using Select, offset for auto increment column
	Offset int
	// contains filtered or unexported fields
}

Generate represents an auto-increment generator for the insert operation.

type GroupBy

type GroupBy struct {
	Inner 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

	// points to the column on the same aggregator
	ColOffset int
	WSOffset  int
}

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 NewGroupBy added in v0.17.0

func NewGroupBy(inner sqlparser.Expr) GroupBy

NewGroupBy creates a new group by from the given fields.

func (GroupBy) AsOrderBy

func (b GroupBy) AsOrderBy() OrderBy

type HashJoin added in v0.19.0

type HashJoin struct {
	LHS, RHS Operator

	// LeftJoin will be true in the case of an outer join
	LeftJoin bool

	// Before offset planning
	JoinComparisons []Comparison

	// Columns stores the column indexes of the columns coming from the left and right side
	// negative value comes from LHS and positive from RHS
	ColumnOffsets []int

	// These are the values that will be hashed together
	LHSKeys, RHSKeys []int
	// contains filtered or unexported fields
}

func NewHashJoin added in v0.19.0

func NewHashJoin(lhs, rhs Operator, outerJoin bool) *HashJoin

func (*HashJoin) AddColumn added in v0.19.0

func (hj *HashJoin) AddColumn(ctx *plancontext.PlanningContext, reuseExisting bool, addToGroupBy bool, expr *sqlparser.AliasedExpr) int

func (*HashJoin) AddJoinPredicate added in v0.19.0

func (hj *HashJoin) AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr)

func (*HashJoin) AddPredicate added in v0.19.0

func (hj *HashJoin) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*HashJoin) Clone added in v0.19.0

func (hj *HashJoin) Clone(inputs []Operator) Operator

func (*HashJoin) FindCol added in v0.19.0

func (hj *HashJoin) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, _ bool) int

func (*HashJoin) GetColumns added in v0.19.0

func (*HashJoin) GetLHS added in v0.19.0

func (hj *HashJoin) GetLHS() Operator

func (*HashJoin) GetOrdering added in v0.19.0

func (hj *HashJoin) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*HashJoin) GetRHS added in v0.19.0

func (hj *HashJoin) GetRHS() Operator

func (*HashJoin) GetSelectExprs added in v0.19.0

func (hj *HashJoin) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*HashJoin) Inputs added in v0.19.0

func (hj *HashJoin) Inputs() []Operator

func (*HashJoin) IsInner added in v0.19.0

func (hj *HashJoin) IsInner() bool

func (*HashJoin) JoinPredicate added in v0.19.0

func (hj *HashJoin) JoinPredicate() sqlparser.Expr

JoinPredicate produces an AST representation of the join condition this join has

func (*HashJoin) MakeInner added in v0.19.0

func (hj *HashJoin) MakeInner()

func (*HashJoin) SetInputs added in v0.19.0

func (hj *HashJoin) SetInputs(operators []Operator)

func (*HashJoin) SetLHS added in v0.19.0

func (hj *HashJoin) SetLHS(op Operator)

func (*HashJoin) SetRHS added in v0.19.0

func (hj *HashJoin) SetRHS(op Operator)

func (*HashJoin) ShortDescription added in v0.19.0

func (hj *HashJoin) ShortDescription() string

type Horizon

type Horizon struct {
	Source Operator

	// If this is a derived table, the two following fields will contain the tableID and name of it
	TableId       *semantics.TableSet
	Alias         string
	ColumnAliases sqlparser.Columns // derived tables can have their column aliases specified outside the subquery

	// QP contains the QueryProjection for this op
	QP *QueryProjection

	Query sqlparser.SelectStatement

	// Columns needed to feed other plans
	Columns       []*sqlparser.ColName
	ColumnsOffset []int
}

Horizon is an operator that allows us to postpone planning things like SELECT/GROUP BY/ORDER BY/LIMIT until later. It contains information about the planning we have to do after deciding how we will send the query to the tablets. If we are able to push down the Horizon under a route, we don't have to plan these things separately and can just copy over the AST constructs to the query being sent to a tablet. If we are not able to push it down, this operator needs to be split up into smaller Project/Aggregate/Sort/Limit operations, some which can be pushed down, and some that have to be evaluated at the vtgate level.

func (*Horizon) AddColumn

func (h *Horizon) AddColumn(ctx *plancontext.PlanningContext, reuse bool, _ bool, expr *sqlparser.AliasedExpr) int

func (*Horizon) AddPredicate

func (h *Horizon) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Horizon) Clone

func (h *Horizon) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*Horizon) FindCol added in v0.18.0

func (h *Horizon) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Horizon) GetColumns added in v0.17.0

func (h *Horizon) GetColumns(ctx *plancontext.PlanningContext) (exprs []*sqlparser.AliasedExpr)

func (*Horizon) GetOrdering added in v0.17.0

func (h *Horizon) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*Horizon) GetSelectExprs added in v0.18.0

func (*Horizon) Inputs

func (h *Horizon) Inputs() []Operator

Inputs implements the Operator interface

func (*Horizon) IsDerived added in v0.18.0

func (h *Horizon) IsDerived() bool

func (*Horizon) IsMergeable added in v0.18.0

func (h *Horizon) 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

func (*Horizon) SetInputs added in v0.17.0

func (h *Horizon) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Horizon) ShortDescription added in v0.17.0

func (h *Horizon) ShortDescription() string

type InfoSchemaRouting added in v0.17.0

type InfoSchemaRouting struct {
	SysTableTableSchema []sqlparser.Expr
	SysTableTableName   map[string]sqlparser.Expr
	Table               *QueryTable
}

InfoSchemaRouting used for information_schema queries. They are special because we usually don't know at plan-time what keyspace the query go to, because we don't see normalized literal values

func (*InfoSchemaRouting) Clone added in v0.17.0

func (isr *InfoSchemaRouting) Clone() Routing

func (*InfoSchemaRouting) Cost added in v0.17.0

func (isr *InfoSchemaRouting) Cost() int

func (*InfoSchemaRouting) Keyspace added in v0.17.0

func (isr *InfoSchemaRouting) Keyspace() *vindexes.Keyspace

func (*InfoSchemaRouting) OpCode added in v0.17.0

func (isr *InfoSchemaRouting) OpCode() engine.Opcode

func (*InfoSchemaRouting) UpdateRoutingParams added in v0.17.0

func (isr *InfoSchemaRouting) UpdateRoutingParams(ctx *plancontext.PlanningContext, rp *engine.RoutingParameters)

type Insert added in v0.17.0

type Insert struct {
	// VTable represents the target table for the insert operation.
	VTable *vindexes.Table
	// AST represents the insert statement from the SQL syntax.
	AST *sqlparser.Insert

	// AutoIncrement represents the auto-increment generator for the insert operation.
	AutoIncrement *Generate
	// Ignore specifies whether to ignore duplicate key errors during insertion.
	Ignore bool

	// ColVindexes are the vindexes that will use the VindexValues or VindexValueOffset
	ColVindexes []*vindexes.ColumnVindex

	// VindexValues specifies values for all the vindex columns.
	VindexValues [][][]evalengine.Expr

	// VindexValueOffset stores the offset for each column in the ColumnVindex
	// that will appear in the result set of the select query.
	VindexValueOffset [][]int
	// contains filtered or unexported fields
}

Insert represents an insert operation on a table.

func (Insert) AddColumn added in v0.17.0

func (Insert) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (Insert) AddPredicate added in v0.17.0

func (Insert) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*Insert) Clone added in v0.17.0

func (i *Insert) Clone([]Operator) Operator

func (Insert) FindCol added in v0.18.0

func (Insert) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (Insert) GetColumns added in v0.17.0

func (Insert) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Insert) GetOrdering added in v0.17.0

func (i *Insert) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (Insert) GetSelectExprs added in v0.18.0

func (Insert) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (Insert) Inputs added in v0.17.0

func (Insert) Inputs() []Operator

Inputs implements the Operator interface

func (Insert) SetInputs added in v0.17.0

func (Insert) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Insert) ShortDescription added in v0.17.0

func (i *Insert) ShortDescription() string

func (*Insert) Statement added in v0.18.0

func (i *Insert) Statement() sqlparser.Statement

func (*Insert) TablesUsed added in v0.17.0

func (i *Insert) TablesUsed() []string

type InsertSelection added in v0.19.0

type InsertSelection struct {
	Select Operator
	Insert Operator

	// ForceNonStreaming when true, select first then insert, this is to avoid locking rows by select for insert.
	ForceNonStreaming bool
	// contains filtered or unexported fields
}

InsertSelection operator represents an INSERT into SELECT FROM query. It holds the operators for running the selection and insertion.

func (InsertSelection) AddColumn added in v0.19.0

func (InsertSelection) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (InsertSelection) AddPredicate added in v0.19.0

func (InsertSelection) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*InsertSelection) Clone added in v0.19.0

func (is *InsertSelection) Clone(inputs []Operator) Operator

func (InsertSelection) FindCol added in v0.19.0

func (InsertSelection) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (InsertSelection) GetColumns added in v0.19.0

func (InsertSelection) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*InsertSelection) GetOrdering added in v0.19.0

func (is *InsertSelection) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (InsertSelection) GetSelectExprs added in v0.19.0

func (InsertSelection) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*InsertSelection) Inputs added in v0.19.0

func (is *InsertSelection) Inputs() []Operator

func (*InsertSelection) SetInputs added in v0.19.0

func (is *InsertSelection) SetInputs(inputs []Operator)

func (*InsertSelection) ShortDescription added in v0.19.0

func (is *InsertSelection) ShortDescription() string

type Join

type Join struct {
	LHS, RHS  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

AddColumn implements the Operator interface

func (*Join) AddJoinPredicate

func (j *Join) AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr)

func (*Join) AddPredicate

func (j *Join) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Join) Clone

func (j *Join) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*Join) Compact

func (j *Join) Compact(ctx *plancontext.PlanningContext) (Operator, *ApplyResult)

func (Join) FindCol added in v0.18.0

func (Join) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (Join) GetColumns added in v0.17.0

func (Join) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Join) GetLHS

func (j *Join) GetLHS() Operator

func (*Join) GetOrdering added in v0.17.0

func (j *Join) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (*Join) GetRHS

func (j *Join) GetRHS() Operator

func (Join) GetSelectExprs added in v0.18.0

func (Join) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*Join) Inputs

func (j *Join) Inputs() []Operator

Inputs implements the Operator interface

func (*Join) IsInner

func (j *Join) IsInner() bool

func (*Join) MakeInner

func (j *Join) MakeInner()

func (*Join) SetInputs added in v0.17.0

func (j *Join) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Join) SetLHS

func (j *Join) SetLHS(operator Operator)

func (*Join) SetRHS

func (j *Join) SetRHS(operator Operator)

func (*Join) ShortDescription added in v0.17.0

func (j *Join) ShortDescription() string

type JoinOp

type JoinOp interface {
	Operator
	GetLHS() Operator
	GetRHS() Operator
	SetLHS(Operator)
	SetRHS(Operator)
	MakeInner()
	IsInner() bool
	AddJoinPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr)
}

type Limit added in v0.17.0

type Limit struct {
	Source Operator
	AST    *sqlparser.Limit

	// Pushed marks whether the limit has been pushed down to the inputs but still need to keep the operator around.
	// For example, `select * from user order by id limit 10`. Even after we push the limit to the route, we need a limit on top
	// since it is a scatter.
	Pushed bool
}

func (*Limit) AddColumn added in v0.17.0

func (l *Limit) AddColumn(ctx *plancontext.PlanningContext, reuse bool, gb bool, expr *sqlparser.AliasedExpr) int

func (*Limit) AddPredicate added in v0.17.0

func (l *Limit) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Limit) Clone added in v0.17.0

func (l *Limit) Clone(inputs []Operator) Operator

func (*Limit) FindCol added in v0.18.0

func (l *Limit) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Limit) GetColumns added in v0.17.0

func (l *Limit) GetColumns(ctx *plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Limit) GetOrdering added in v0.17.0

func (l *Limit) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*Limit) GetSelectExprs added in v0.18.0

func (l *Limit) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*Limit) Inputs added in v0.17.0

func (l *Limit) Inputs() []Operator

func (*Limit) SetInputs added in v0.17.0

func (l *Limit) SetInputs(operators []Operator)

func (*Limit) ShortDescription added in v0.17.0

func (l *Limit) ShortDescription() string

type LockAndComment added in v0.18.0

type LockAndComment struct {
	Source   Operator
	Comments *sqlparser.ParsedComments
	Lock     sqlparser.Lock
}

LockAndComment contains any comments or locking directives we want on all queries down from this operator

func (*LockAndComment) AddColumn added in v0.18.0

func (l *LockAndComment) AddColumn(ctx *plancontext.PlanningContext, reuseExisting bool, addToGroupBy bool, expr *sqlparser.AliasedExpr) int

func (*LockAndComment) AddPredicate added in v0.18.0

func (l *LockAndComment) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*LockAndComment) Clone added in v0.18.0

func (l *LockAndComment) Clone(inputs []Operator) Operator

func (*LockAndComment) FindCol added in v0.18.0

func (l *LockAndComment) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*LockAndComment) GetColumns added in v0.18.0

func (*LockAndComment) GetOrdering added in v0.18.0

func (l *LockAndComment) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*LockAndComment) GetSelectExprs added in v0.18.0

func (*LockAndComment) Inputs added in v0.18.0

func (l *LockAndComment) Inputs() []Operator

func (*LockAndComment) SetInputs added in v0.18.0

func (l *LockAndComment) SetInputs(operators []Operator)

func (*LockAndComment) ShortDescription added in v0.18.0

func (l *LockAndComment) ShortDescription() string

type NoneRouting added in v0.17.0

type NoneRouting struct {
	// contains filtered or unexported fields
}

NoneRouting is used when we know that this Route will return no results. Can be merged with any other route going to the same keyspace

func (*NoneRouting) Clone added in v0.17.0

func (n *NoneRouting) Clone() Routing

func (*NoneRouting) Cost added in v0.17.0

func (n *NoneRouting) Cost() int

func (*NoneRouting) Keyspace added in v0.17.0

func (n *NoneRouting) Keyspace() *vindexes.Keyspace

func (*NoneRouting) OpCode added in v0.17.0

func (n *NoneRouting) OpCode() engine.Opcode

func (*NoneRouting) UpdateRoutingParams added in v0.17.0

func (n *NoneRouting) UpdateRoutingParams(_ *plancontext.PlanningContext, rp *engine.RoutingParameters)

type Offset added in v0.17.0

type Offset int

Offset is used when we are only passing through data from an incoming column

type OpWithAST added in v0.18.0

type OpWithAST interface {
	Operator
	Statement() sqlparser.Statement
}

type Operator added in v0.19.0

type Operator interface {
	// Clone will return a copy of this operator, protected so changed to the original will not impact the clone
	Clone(inputs []Operator) Operator

	// Inputs returns the inputs for this operator
	Inputs() []Operator

	// SetInputs changes the inputs for this op
	SetInputs([]Operator)

	// AddPredicate is used to push predicates. It pushed it as far down as is possible in the tree.
	// If we encounter a join and the predicate depends on both sides of the join, the predicate will be split into two parts,
	// where data is fetched from the LHS of the join to be used in the evaluation on the RHS
	// TODO: we should remove this and replace it with rewriters
	AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

	AddColumn(ctx *plancontext.PlanningContext, reuseExisting bool, addToGroupBy bool, expr *sqlparser.AliasedExpr) int

	FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

	GetColumns(ctx *plancontext.PlanningContext) []*sqlparser.AliasedExpr
	GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

	ShortDescription() string

	GetOrdering(ctx *plancontext.PlanningContext) []OrderBy
}

Operator forms the tree of operators, representing the declarative query provided. The operator tree is no actually runnable, it's an intermediate representation used while query planning The mental model are operators that pull data from each other, the root being the full query output, and the leaves are most often `Route`s, representing communication with one or more shards. We want to push down as much work as possible under these Routes

func AddPredicate

func AddPredicate(
	ctx *plancontext.PlanningContext,
	join JoinOp,
	expr sqlparser.Expr,
	joinPredicates bool,
	newFilter func(Operator, sqlparser.Expr) Operator,
) Operator

func BottomUp added in v0.19.0

func BottomUp(
	root Operator,
	resolveID func(Operator) semantics.TableSet,
	visit VisitF,
	shouldVisit ShouldVisit,
) Operator

BottomUp rewrites an operator tree from the bottom up. BottomUp applies a transformation function to the given operator tree from the bottom up. Each callback [f] returns a ApplyResult that is aggregated into a final output indicating whether the operator tree was changed.

func Clone

func Clone(op Operator) Operator

func FixedPointBottomUp added in v0.19.0

func FixedPointBottomUp(
	root Operator,
	resolveID func(Operator) semantics.TableSet,
	visit VisitF,
	shouldVisit ShouldVisit,
) (op Operator)

FixedPointBottomUp rewrites an operator tree much like BottomUp does, but does the rewriting repeatedly, until a tree walk is done with no changes to the tree.

func PlanQuery

func PlanQuery(ctx *plancontext.PlanningContext, stmt sqlparser.Statement) (result Operator, err error)

PlanQuery creates a query plan for a given SQL statement

func ToSQL

func TopDown added in v0.19.0

func TopDown(
	root Operator,
	resolveID func(Operator) semantics.TableSet,
	visit VisitF,
	shouldVisit ShouldVisit,
) Operator

TopDown rewrites an operator tree from the bottom up. BottomUp applies a transformation function to the given operator tree from the bottom up. Each callback [f] returns a ApplyResult that is aggregated into a final output indicating whether the operator tree was changed.

Parameters: - root: The root operator of the tree to be traversed. - resolveID: A function to resolve the TableSet of an operator. - visit: The VisitF function to be called for each visited operator. - shouldVisit: The ShouldVisit function to control which nodes and ancestors to visit and which to skip.

Returns: - Operator: The root of the (potentially) transformed operator tree. - error: An error if any occurred during the traversal.

type OrderBy

type OrderBy struct {
	Inner *sqlparser.Order

	// See GroupBy#SimplifiedExpr for more details about this
	SimplifiedExpr sqlparser.Expr
}

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.

func (OrderBy) Map added in v0.19.0

func (ob OrderBy) Map(mappingFunc func(sqlparser.Expr) sqlparser.Expr) OrderBy

Map takes in a mapping function and applies it to both the expression in OrderBy.

type Ordering added in v0.17.0

type Ordering struct {
	Source  Operator
	Offset  []int
	WOffset []int

	Order         []OrderBy
	ResultColumns int
}

func (*Ordering) AddColumn added in v0.17.0

func (o *Ordering) AddColumn(ctx *plancontext.PlanningContext, reuse bool, gb bool, expr *sqlparser.AliasedExpr) int

func (*Ordering) AddPredicate added in v0.17.0

func (o *Ordering) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Ordering) Clone added in v0.17.0

func (o *Ordering) Clone(inputs []Operator) Operator

func (*Ordering) FindCol added in v0.18.0

func (o *Ordering) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Ordering) GetColumns added in v0.17.0

func (o *Ordering) GetColumns(ctx *plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Ordering) GetOrdering added in v0.17.0

func (o *Ordering) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (*Ordering) GetSelectExprs added in v0.18.0

func (o *Ordering) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*Ordering) Inputs added in v0.17.0

func (o *Ordering) Inputs() []Operator

func (*Ordering) SetInputs added in v0.17.0

func (o *Ordering) SetInputs(operators []Operator)

func (*Ordering) ShortDescription added in v0.17.0

func (o *Ordering) ShortDescription() string

type Phase added in v0.18.0

type Phase int
const (
	DONE Phase
)

func (Phase) String added in v0.18.0

func (p Phase) String() string

type ProjCols added in v0.18.0

type ProjCols interface {
	GetColumns() []*sqlparser.AliasedExpr
	GetSelectExprs() sqlparser.SelectExprs
}

ProjCols is used to enable projections that are only valid if we can push them into a route, and we never need to ask it about offsets

type ProjExpr added in v0.17.0

type ProjExpr struct {
	Original *sqlparser.AliasedExpr // this is the expression the user asked for. should only be used to decide on the column alias
	EvalExpr sqlparser.Expr         // EvalExpr is the expression that will be evaluated at runtime
	ColExpr  sqlparser.Expr         // ColExpr is used during planning to figure out which column this ProjExpr is representing
	Info     ExprInfo               // Here we store information about evalengine, offsets or subqueries
}

func (*ProjExpr) String added in v0.18.0

func (pe *ProjExpr) String() string

type Projection added in v0.17.0

type Projection struct {
	Source Operator

	// Columns contain the expressions as viewed from the outside of this operator
	Columns ProjCols

	// DT will hold all the necessary information if this is a derived table projection
	DT       *DerivedTable
	FromAggr bool
}

Projection is used when we need to evaluate expressions on the vtgate It uses the evalengine to accomplish its goal

func (*Projection) AddColumn added in v0.17.0

func (p *Projection) AddColumn(ctx *plancontext.PlanningContext, reuse bool, addToGroupBy bool, ae *sqlparser.AliasedExpr) int

func (*Projection) AddPredicate added in v0.17.0

func (p *Projection) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Projection) AllOffsets added in v0.17.0

func (p *Projection) AllOffsets() (cols []int)

AllOffsets returns a slice of integer offsets for all columns in the Projection if all columns are of type Offset. If any column is not of type Offset, it returns nil.

func (*Projection) Clone added in v0.17.0

func (p *Projection) Clone(inputs []Operator) Operator

func (*Projection) Compact added in v0.17.0

func (*Projection) FindCol added in v0.18.0

func (p *Projection) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Projection) GetAliasedProjections added in v0.18.0

func (p *Projection) GetAliasedProjections() (AliasedProjections, error)

func (*Projection) GetColumns added in v0.17.0

func (*Projection) GetOrdering added in v0.17.0

func (p *Projection) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*Projection) GetSelectExprs added in v0.18.0

func (*Projection) Inputs added in v0.17.0

func (p *Projection) Inputs() []Operator

func (*Projection) SetInputs added in v0.17.0

func (p *Projection) SetInputs(operators []Operator)

func (*Projection) ShortDescription added in v0.17.0

func (p *Projection) ShortDescription() string

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, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (*QueryGraph) AddPredicate

func (qg *QueryGraph) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*QueryGraph) Clone

func (qg *QueryGraph) Clone([]Operator) Operator

Clone implements the Operator interface

func (QueryGraph) FindCol added in v0.18.0

func (QueryGraph) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (QueryGraph) GetColumns added in v0.17.0

func (QueryGraph) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*QueryGraph) GetOrdering added in v0.17.0

func (qg *QueryGraph) GetOrdering(*plancontext.PlanningContext) []OrderBy

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) GetSelectExprs added in v0.18.0

func (QueryGraph) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (QueryGraph) Inputs

func (QueryGraph) Inputs() []Operator

Inputs implements the Operator interface

func (QueryGraph) SetInputs added in v0.17.0

func (QueryGraph) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*QueryGraph) ShortDescription added in v0.17.0

func (qg *QueryGraph) ShortDescription() string

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

	// 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 CreateQPFromSelectStatement added in v0.18.0

func CreateQPFromSelectStatement(ctx *plancontext.PlanningContext, stmt sqlparser.SelectStatement) *QueryProjection

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, allowComplexExpression bool) (out []Aggr, complex bool)

func (*QueryProjection) AlignGroupByAndOrderBy

func (qp *QueryProjection) AlignGroupByAndOrderBy(ctx *plancontext.PlanningContext) bool

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

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) 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) OldAlignGroupByAndOrderBy added in v0.17.0

func (qp *QueryProjection) OldAlignGroupByAndOrderBy(ctx *plancontext.PlanningContext)

OldAlignGroupByAndOrderBy TODO Remove once all of horizon planning is done on the operators

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 Rewrite added in v0.19.0

type Rewrite struct {
	Message string
}

type Route

type Route struct {
	Source Operator

	// Routes that have been merged into this one.
	MergedWith []*Route

	Routing Routing

	Ordering []RouteOrdering

	Comments *sqlparser.ParsedComments
	Lock     sqlparser.Lock

	ResultColumns int
}

func (*Route) AddColumn

func (r *Route) AddColumn(ctx *plancontext.PlanningContext, reuse bool, gb bool, expr *sqlparser.AliasedExpr) int

func (*Route) AddPredicate

func (r *Route) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Route) Clone

func (r *Route) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*Route) Cost

func (r *Route) Cost() int

Cost implements the Operator interface

func (*Route) FindCol added in v0.18.0

func (r *Route) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, _ bool) int

func (*Route) GetColumns added in v0.17.0

func (r *Route) GetColumns(ctx *plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Route) GetOrdering added in v0.17.0

func (r *Route) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*Route) GetSelectExprs added in v0.18.0

func (r *Route) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*Route) Inputs

func (r *Route) Inputs() []Operator

Inputs implements the Operator interface

func (*Route) IsSingleShard

func (r *Route) IsSingleShard() bool

func (*Route) IsSingleShardOrByDestination added in v0.19.0

func (r *Route) IsSingleShardOrByDestination() bool

func (*Route) SetInputs added in v0.17.0

func (r *Route) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Route) ShortDescription added in v0.17.0

func (r *Route) ShortDescription() string

func (*Route) TablesUsed

func (r *Route) TablesUsed() []string

TablesUsed returns tables used by MergedWith routes, which are not included in Inputs() and thus not a part of the operator tree

type RouteOrdering added in v0.17.0

type RouteOrdering struct {
	AST sqlparser.Expr
	// Offset and WOffset will contain the offset to the column (and the weightstring column). -1 if it's missing
	Offset, WOffset int
	Direction       sqlparser.OrderDirection
}

type Routing added in v0.17.0

type Routing interface {
	// UpdateRoutingParams allows a Routing to control the routing params that will be used by the engine Route
	// OpCode is already set, and the default keyspace is set for read queries
	UpdateRoutingParams(ctx *plancontext.PlanningContext, rp *engine.RoutingParameters)

	// Clone returns a copy of the routing. Since we are trying different variation of merging,
	// one Routing can be used in different constellations.
	// We don't want these different alternatives to influence each other, and cloning allows this
	Clone() Routing

	// Cost returns the cost of this Route.
	Cost() int
	OpCode() engine.Opcode
	Keyspace() *vindexes.Keyspace // note that all routings do not have a keyspace, so this method can return nil
	// contains filtered or unexported methods
}

Routing is used for the routing and merging logic of `Route`s. Every Route has a Routing object, and this object is updated when predicates are found, and when merging `Route`s together

func UpdateRoutingLogic added in v0.17.0

func UpdateRoutingLogic(ctx *plancontext.PlanningContext, expr sqlparser.Expr, r Routing) Routing

UpdateRoutingLogic first checks if we are dealing with a predicate that

type SelectExpr

type SelectExpr struct {
	Col  sqlparser.SelectExpr
	Aggr bool
}

SelectExpr provides whether the column 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.

func (SelectExpr) GetExpr

func (s SelectExpr) GetExpr() (sqlparser.Expr, error)

GetExpr returns the underlying sqlparser.Expr of our SelectExpr

type SequenceRouting added in v0.17.0

type SequenceRouting struct {
	// contains filtered or unexported fields
}

func (*SequenceRouting) Clone added in v0.17.0

func (sr *SequenceRouting) Clone() Routing

func (*SequenceRouting) Cost added in v0.17.0

func (sr *SequenceRouting) Cost() int

func (*SequenceRouting) Keyspace added in v0.17.0

func (sr *SequenceRouting) Keyspace() *vindexes.Keyspace

func (*SequenceRouting) OpCode added in v0.17.0

func (sr *SequenceRouting) OpCode() engine.Opcode

func (*SequenceRouting) UpdateRoutingParams added in v0.17.0

func (sr *SequenceRouting) UpdateRoutingParams(_ *plancontext.PlanningContext, rp *engine.RoutingParameters)

type Sequential added in v0.19.0

type Sequential struct {
	Sources []Operator
	// contains filtered or unexported fields
}

func (Sequential) AddColumn added in v0.19.0

func (Sequential) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (Sequential) AddPredicate added in v0.19.0

func (Sequential) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*Sequential) Clone added in v0.19.0

func (s *Sequential) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (Sequential) FindCol added in v0.19.0

func (Sequential) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (Sequential) GetColumns added in v0.19.0

func (Sequential) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Sequential) GetOrdering added in v0.19.0

func (s *Sequential) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (Sequential) GetSelectExprs added in v0.19.0

func (Sequential) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*Sequential) Inputs added in v0.19.0

func (s *Sequential) Inputs() []Operator

Inputs implements the Operator interface

func (*Sequential) SetInputs added in v0.19.0

func (s *Sequential) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Sequential) ShortDescription added in v0.19.0

func (s *Sequential) ShortDescription() string

type SetExpr added in v0.18.0

type SetExpr struct {
	Name *sqlparser.ColName
	Expr *ProjExpr
}

func (SetExpr) String added in v0.18.0

func (se SetExpr) String() string

type ShardedRouting added in v0.17.0

type ShardedRouting struct {
	// 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

	RouteOpCode engine.Opcode

	// 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
	// contains filtered or unexported fields
}

ShardedRouting is what we use for all tables that exist in a sharded keyspace It knows about available vindexes and can use them for routing when applicable

func (*ShardedRouting) Clone added in v0.17.0

func (tr *ShardedRouting) Clone() Routing

func (*ShardedRouting) Cost added in v0.17.0

func (tr *ShardedRouting) Cost() int

func (*ShardedRouting) Keyspace added in v0.17.0

func (tr *ShardedRouting) Keyspace() *vindexes.Keyspace

func (*ShardedRouting) OpCode added in v0.17.0

func (tr *ShardedRouting) OpCode() engine.Opcode

func (*ShardedRouting) PickBestAvailableVindex added in v0.17.0

func (tr *ShardedRouting) PickBestAvailableVindex()

PickBestAvailableVindex goes over the available vindexes for this route and picks the best one available.

func (*ShardedRouting) SelectedVindex added in v0.17.0

func (tr *ShardedRouting) SelectedVindex() vindexes.Vindex

func (*ShardedRouting) UpdateRoutingParams added in v0.17.0

func (tr *ShardedRouting) UpdateRoutingParams(_ *plancontext.PlanningContext, rp *engine.RoutingParameters)

func (*ShardedRouting) VindexExpressions added in v0.17.0

func (tr *ShardedRouting) VindexExpressions() []sqlparser.Expr

type ShouldVisit added in v0.19.0

type ShouldVisit func(Operator) VisitRule

ShouldVisit is used when we want to control which nodes and ancestors to visit and which to skip

type StarProjections added in v0.18.0

type StarProjections sqlparser.SelectExprs

Used when there are stars in the expressions that we were unable to expand

func (StarProjections) GetColumns added in v0.18.0

func (sp StarProjections) GetColumns() []*sqlparser.AliasedExpr

func (StarProjections) GetSelectExprs added in v0.18.0

func (sp StarProjections) GetSelectExprs() sqlparser.SelectExprs

type SubQuery

type SubQuery struct {
	// Fields filled in at the time of construction:
	Outer      Operator             // Outer query operator.
	Subquery   Operator             // Subquery operator.
	FilterType opcode.PulloutOpcode // Type of subquery filter.
	Original   sqlparser.Expr       // This is the expression we should use if we can merge the inner to the outer

	Predicates        sqlparser.Exprs   // Predicates joining outer and inner queries. Empty for uncorrelated subqueries.
	OuterPredicate    sqlparser.Expr    // This is the predicate that is using the subquery expression. It will not be empty for projections
	ArgName           string            // This is the name of the ColName or Argument used to replace the subquery
	TopLevel          bool              // will be false if the subquery is deeply nested
	JoinColumns       []applyJoinColumn // Broken up join predicates.
	SubqueryValueName string            // Value name returned by the subquery (uncorrelated queries).
	HasValuesName     string            // Argument name passed to the subquery (uncorrelated queries).

	// Fields related to correlated subqueries:
	Vars map[string]int // Arguments copied from outer to inner, set during offset planning.

	IsProjection bool
	// contains filtered or unexported fields
}

SubQuery represents a subquery used for filtering rows in an outer query through a join.

func (*SubQuery) AddColumn

func (sq *SubQuery) AddColumn(ctx *plancontext.PlanningContext, reuseExisting bool, addToGroupBy bool, exprs *sqlparser.AliasedExpr) int

func (*SubQuery) AddPredicate

func (sq *SubQuery) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*SubQuery) Clone

func (sq *SubQuery) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*SubQuery) FindCol added in v0.18.0

func (sq *SubQuery) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*SubQuery) GetColumns added in v0.17.0

func (sq *SubQuery) GetColumns(ctx *plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*SubQuery) GetJoinColumns added in v0.18.0

func (sq *SubQuery) GetJoinColumns(ctx *plancontext.PlanningContext, outer Operator) ([]applyJoinColumn, error)

func (*SubQuery) GetMergePredicates added in v0.18.0

func (sq *SubQuery) GetMergePredicates() []sqlparser.Expr

GetMergePredicates returns the predicates that we can use to try to merge this subquery with the outer query.

func (*SubQuery) GetOrdering added in v0.17.0

func (sq *SubQuery) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*SubQuery) GetSelectExprs added in v0.18.0

func (sq *SubQuery) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*SubQuery) Inputs

func (sq *SubQuery) Inputs() []Operator

Inputs implements the Operator interface

func (*SubQuery) OuterExpressionsNeeded added in v0.18.0

func (sq *SubQuery) OuterExpressionsNeeded(ctx *plancontext.PlanningContext, outer Operator) (result []*sqlparser.ColName)

func (*SubQuery) SetInputs added in v0.17.0

func (sq *SubQuery) SetInputs(inputs []Operator)

SetInputs implements the Operator interface

func (*SubQuery) ShortDescription added in v0.17.0

func (sq *SubQuery) ShortDescription() string

type SubQueryBuilder added in v0.18.0

type SubQueryBuilder struct {
	Inner []*SubQuery
	// contains filtered or unexported fields
}

type SubQueryContainer added in v0.18.0

type SubQueryContainer struct {
	Outer Operator
	Inner []*SubQuery
}

SubQueryContainer stores the information about a query and it's subqueries. The inner subqueries can be executed in any order, so we store them like this so we can see more opportunities for merging

func (*SubQueryContainer) AddColumn added in v0.18.0

func (sqc *SubQueryContainer) AddColumn(ctx *plancontext.PlanningContext, reuseExisting bool, addToGroupBy bool, exprs *sqlparser.AliasedExpr) int

func (*SubQueryContainer) AddPredicate added in v0.18.0

func (sqc *SubQueryContainer) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*SubQueryContainer) Clone added in v0.18.0

func (sqc *SubQueryContainer) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*SubQueryContainer) FindCol added in v0.18.0

func (sqc *SubQueryContainer) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*SubQueryContainer) GetColumns added in v0.18.0

func (*SubQueryContainer) GetOrdering added in v0.18.0

func (sqc *SubQueryContainer) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (*SubQueryContainer) GetSelectExprs added in v0.18.0

func (*SubQueryContainer) Inputs added in v0.18.0

func (sqc *SubQueryContainer) Inputs() []Operator

Inputs implements the Operator interface

func (*SubQueryContainer) SetInputs added in v0.18.0

func (sqc *SubQueryContainer) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*SubQueryContainer) ShortDescription added in v0.18.0

func (sqc *SubQueryContainer) ShortDescription() string

type SubQueryExpression added in v0.18.0

type SubQueryExpression []*SubQuery

type Table

type Table struct {
	QTable  *QueryTable
	VTable  *vindexes.Table
	Columns []*sqlparser.ColName
	// contains filtered or unexported fields
}

func (*Table) AddCol

func (to *Table) AddCol(col *sqlparser.ColName)

func (*Table) AddColumn

func (*Table) AddPredicate

func (to *Table) AddPredicate(_ *plancontext.PlanningContext, expr sqlparser.Expr) Operator

AddPredicate implements the PhysicalOperator interface

func (*Table) Clone

func (to *Table) Clone([]Operator) Operator

Clone implements the Operator interface

func (*Table) FindCol added in v0.18.0

func (to *Table) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Table) GetColNames added in v0.17.0

func (to *Table) GetColNames() []*sqlparser.ColName

func (*Table) GetColumns

func (*Table) GetOrdering added in v0.17.0

func (to *Table) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (*Table) GetSelectExprs added in v0.18.0

func (to *Table) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (Table) Inputs

func (Table) Inputs() []Operator

Inputs implements the Operator interface

func (Table) SetInputs added in v0.17.0

func (Table) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Table) ShortDescription added in v0.17.0

func (to *Table) ShortDescription() string

func (*Table) TablesUsed

func (to *Table) TablesUsed() []string

type TableUser

type TableUser interface {
	TablesUsed() []string
}

TableUser is used to signal that this operator directly interacts with one or more tables

type TargetTable added in v0.19.0

type TargetTable struct {
	ID     semantics.TableSet
	VTable *vindexes.Table
	Name   sqlparser.TableName
}

type TargetedRouting added in v0.17.0

type TargetedRouting struct {

	// targetDestination specifies an explicit target destination tablet type
	TargetDestination key.Destination
	// contains filtered or unexported fields
}

TargetedRouting is used when the user has used syntax to target the Route against a specific set of shards and/or tablet type. Can't be merged with anything else.

func (*TargetedRouting) Clone added in v0.17.0

func (tr *TargetedRouting) Clone() Routing

func (*TargetedRouting) Cost added in v0.17.0

func (tr *TargetedRouting) Cost() int

func (*TargetedRouting) Keyspace added in v0.17.0

func (tr *TargetedRouting) Keyspace() *vindexes.Keyspace

func (*TargetedRouting) OpCode added in v0.17.0

func (tr *TargetedRouting) OpCode() engine.Opcode

func (*TargetedRouting) UpdateRoutingParams added in v0.17.0

func (tr *TargetedRouting) UpdateRoutingParams(_ *plancontext.PlanningContext, rp *engine.RoutingParameters)

type Union

type Union struct {
	Sources []Operator

	// These are the select expressions coming from each source
	Selects []sqlparser.SelectExprs
	// contains filtered or unexported fields
}

func (*Union) AddColumn

func (u *Union) AddColumn(ctx *plancontext.PlanningContext, reuse bool, gb bool, expr *sqlparser.AliasedExpr) int

func (*Union) AddPredicate

func (u *Union) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

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) Clone

func (u *Union) Clone(inputs []Operator) Operator

Clone implements the Operator interface

func (*Union) FindCol added in v0.18.0

func (u *Union) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Union) GetColumns added in v0.17.0

func (u *Union) GetColumns(ctx *plancontext.PlanningContext) (result []*sqlparser.AliasedExpr)

func (*Union) GetOrdering added in v0.17.0

func (u *Union) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (*Union) GetSelectExprs added in v0.18.0

func (u *Union) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (*Union) GetSelectFor

func (u *Union) GetSelectFor(source int) *sqlparser.Select

func (*Union) Inputs

func (u *Union) Inputs() []Operator

Inputs implements the Operator interface

func (*Union) NoLHSTableSet added in v0.17.0

func (u *Union) NoLHSTableSet()

func (*Union) SetInputs added in v0.17.0

func (u *Union) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Union) ShortDescription added in v0.17.0

func (u *Union) ShortDescription() string

type Update

type Update struct {
	QTable              *QueryTable
	VTable              *vindexes.Table
	Assignments         []SetExpr
	ChangedVindexValues map[string]*engine.VindexValues
	OwnedVindexQuery    string
	Ignore              sqlparser.Ignore
	OrderBy             sqlparser.OrderBy
	Limit               *sqlparser.Limit

	// these subqueries cannot be merged as they are part of the changed vindex values
	// these values are needed to be sent over to lookup vindex for update.
	// On merging this information will be lost, so subquery merge is blocked.
	SubQueriesArgOnChangedVindex []string
	// contains filtered or unexported fields
}

func (Update) AddColumn

func (Update) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (Update) AddPredicate

func (Update) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*Update) Clone

func (u *Update) Clone([]Operator) Operator

Clone implements the Operator interface

func (Update) FindCol added in v0.18.0

func (Update) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (Update) GetColumns added in v0.17.0

func (Update) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Update) GetOrdering added in v0.17.0

func (u *Update) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (Update) GetSelectExprs added in v0.18.0

func (Update) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (Update) Inputs

func (Update) Inputs() []Operator

Inputs implements the Operator interface

func (Update) SetInputs added in v0.17.0

func (Update) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Update) ShortDescription added in v0.17.0

func (u *Update) ShortDescription() string

func (*Update) TablesUsed

func (u *Update) TablesUsed() []string

type Upsert added in v0.19.0

type Upsert struct {
	Sources []UpsertSource
	// contains filtered or unexported fields
}

Upsert represents an insert on duplicate key operation on a table.

func (Upsert) AddColumn added in v0.19.0

func (Upsert) AddColumn(*plancontext.PlanningContext, bool, bool, *sqlparser.AliasedExpr) int

AddColumn implements the Operator interface

func (Upsert) AddPredicate added in v0.19.0

func (Upsert) AddPredicate(*plancontext.PlanningContext, sqlparser.Expr) Operator

AddPredicate implements the Operator interface

func (*Upsert) Clone added in v0.19.0

func (u *Upsert) Clone(inputs []Operator) Operator

func (Upsert) FindCol added in v0.19.0

func (Upsert) FindCol(*plancontext.PlanningContext, sqlparser.Expr, bool) int

func (Upsert) GetColumns added in v0.19.0

func (Upsert) GetColumns(*plancontext.PlanningContext) []*sqlparser.AliasedExpr

func (*Upsert) GetOrdering added in v0.19.0

func (u *Upsert) GetOrdering(ctx *plancontext.PlanningContext) []OrderBy

func (Upsert) GetSelectExprs added in v0.19.0

func (Upsert) GetSelectExprs(*plancontext.PlanningContext) sqlparser.SelectExprs

func (*Upsert) Inputs added in v0.19.0

func (u *Upsert) Inputs() []Operator

func (*Upsert) SetInputs added in v0.19.0

func (u *Upsert) SetInputs(inputs []Operator)

func (*Upsert) ShortDescription added in v0.19.0

func (u *Upsert) ShortDescription() string

type UpsertSource added in v0.19.0

type UpsertSource struct {
	Insert Operator
	Update Operator
}

type VerifyOp added in v0.18.0

type VerifyOp struct {
	Op  Operator
	Typ string
}

VerifyOp keeps the information about the foreign key verification operation. It is a Parent verification or a Child verification.

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) AddCol

func (v *Vindex) AddCol(col *sqlparser.ColName)

func (*Vindex) AddColumn

func (v *Vindex) AddColumn(ctx *plancontext.PlanningContext, reuse bool, gb bool, ae *sqlparser.AliasedExpr) int

func (*Vindex) AddPredicate

func (v *Vindex) AddPredicate(ctx *plancontext.PlanningContext, expr sqlparser.Expr) Operator

func (*Vindex) CheckValid

func (v *Vindex) CheckValid()

func (*Vindex) Clone

func (v *Vindex) Clone([]Operator) Operator

Clone implements the Operator interface

func (*Vindex) FindCol added in v0.18.0

func (v *Vindex) FindCol(ctx *plancontext.PlanningContext, expr sqlparser.Expr, underRoute bool) int

func (*Vindex) GetColNames added in v0.17.0

func (v *Vindex) GetColNames() []*sqlparser.ColName

func (*Vindex) GetColumns

func (*Vindex) GetOrdering added in v0.17.0

func (v *Vindex) GetOrdering(*plancontext.PlanningContext) []OrderBy

func (*Vindex) GetSelectExprs added in v0.18.0

func (v *Vindex) GetSelectExprs(ctx *plancontext.PlanningContext) sqlparser.SelectExprs

func (Vindex) Inputs

func (Vindex) Inputs() []Operator

Inputs implements the Operator interface

func (Vindex) SetInputs added in v0.17.0

func (Vindex) SetInputs(ops []Operator)

SetInputs implements the Operator interface

func (*Vindex) ShortDescription added in v0.17.0

func (v *Vindex) ShortDescription() string

func (*Vindex) TablesUsed

func (v *Vindex) TablesUsed() []string

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

type VindexTable

type VindexTable struct {
	TableID    semantics.TableSet
	Alias      *sqlparser.AliasedTableExpr
	Table      sqlparser.TableName
	Predicates []sqlparser.Expr
	VTable     *vindexes.Table
}

VindexTable contains information about the vindex table we want to query

type VisitF added in v0.19.0

type VisitF func(
	op Operator,
	lhsTables semantics.TableSet,
	isRoot bool,
) (Operator, *ApplyResult)

VisitF is the visitor that walks an operator tree

type VisitRule added in v0.19.0

type VisitRule bool

VisitRule signals to the rewriter if the children of this operator should be visited or not

const (
	VisitChildren VisitRule = true
	SkipChildren  VisitRule = false
)

Jump to

Keyboard shortcuts

? : This menu
/ : Search site
f or F : Jump to
y or Y : Canonical URL