Skip to content

Commit

Permalink
Merge pull request #12575 from knz/set-needed-cols
Browse files Browse the repository at this point in the history
sql: rewrite setNeededColumns() and complete the feature.
  • Loading branch information
knz authored Dec 25, 2016
2 parents 0efe656 + a249028 commit 789f749
Show file tree
Hide file tree
Showing 37 changed files with 604 additions and 123 deletions.
1 change: 0 additions & 1 deletion pkg/sql/alter_table.go
Original file line number Diff line number Diff line change
Expand Up @@ -402,7 +402,6 @@ func (n *alterTableNode) Ordering() orderingInfo { return orderingInfo{} }
func (n *alterTableNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *alterTableNode) DebugValues() debugValues { return debugValues{} }
func (n *alterTableNode) SetLimitHint(_ int64, _ bool) {}
func (n *alterTableNode) setNeededColumns(_ []bool) {}
func (n *alterTableNode) MarkDebug(mode explainMode) {}

func applyColumnMutation(
Expand Down
1 change: 0 additions & 1 deletion pkg/sql/copy.go
Original file line number Diff line number Diff line change
Expand Up @@ -53,7 +53,6 @@ func (n *copyNode) Columns() ResultColumns { return n.resultColumns }
func (*copyNode) Ordering() orderingInfo { return orderingInfo{} }
func (*copyNode) Values() parser.DTuple { return nil }
func (*copyNode) SetLimitHint(_ int64, _ bool) {}
func (*copyNode) setNeededColumns(_ []bool) {}
func (*copyNode) MarkDebug(_ explainMode) {}
func (*copyNode) expandPlan() error { return nil }
func (*copyNode) Next() (bool, error) { return false, nil }
Expand Down
5 changes: 0 additions & 5 deletions pkg/sql/create.go
Original file line number Diff line number Diff line change
Expand Up @@ -143,7 +143,6 @@ func (n *createDatabaseNode) Ordering() orderingInfo { return orderingInfo
func (n *createDatabaseNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *createDatabaseNode) DebugValues() debugValues { return debugValues{} }
func (n *createDatabaseNode) SetLimitHint(_ int64, _ bool) {}
func (n *createDatabaseNode) setNeededColumns(_ []bool) {}
func (n *createDatabaseNode) MarkDebug(mode explainMode) {}

type createIndexNode struct {
Expand Down Expand Up @@ -259,7 +258,6 @@ func (n *createIndexNode) Ordering() orderingInfo { return orderingInfo{}
func (n *createIndexNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *createIndexNode) DebugValues() debugValues { return debugValues{} }
func (n *createIndexNode) SetLimitHint(_ int64, _ bool) {}
func (n *createIndexNode) setNeededColumns(_ []bool) {}
func (n *createIndexNode) MarkDebug(mode explainMode) {}

type createUserNode struct {
Expand Down Expand Up @@ -348,7 +346,6 @@ func (n *createUserNode) Ordering() orderingInfo { return orderingInfo{} }
func (n *createUserNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *createUserNode) DebugValues() debugValues { return debugValues{} }
func (n *createUserNode) SetLimitHint(_ int64, _ bool) {}
func (n *createUserNode) setNeededColumns(_ []bool) {}
func (n *createUserNode) MarkDebug(mode explainMode) {}

type createViewNode struct {
Expand Down Expand Up @@ -498,7 +495,6 @@ func (n *createViewNode) Columns() ResultColumns { return make(ResultColum
func (n *createViewNode) Ordering() orderingInfo { return orderingInfo{} }
func (n *createViewNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *createViewNode) DebugValues() debugValues { return debugValues{} }
func (n *createViewNode) setNeededColumns(_ []bool) {}
func (n *createViewNode) SetLimitHint(_ int64, _ bool) {}
func (n *createViewNode) MarkDebug(mode explainMode) {}

Expand Down Expand Up @@ -729,7 +725,6 @@ func (n *createTableNode) Ordering() orderingInfo { return orderingInfo{}
func (n *createTableNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *createTableNode) DebugValues() debugValues { return debugValues{} }
func (n *createTableNode) SetLimitHint(_ int64, _ bool) {}
func (n *createTableNode) setNeededColumns(_ []bool) {}
func (n *createTableNode) MarkDebug(mode explainMode) {}

type indexMatch bool
Expand Down
1 change: 0 additions & 1 deletion pkg/sql/delayed.go
Original file line number Diff line number Diff line change
Expand Up @@ -32,7 +32,6 @@ type delayedNode struct {
type nodeConstructor func(p *planner) (planNode, error)

func (d *delayedNode) SetLimitHint(_ int64, _ bool) {}
func (d *delayedNode) setNeededColumns(_ []bool) {}

func (d *delayedNode) expandPlan() error {
v, err := d.constructor(d.p)
Expand Down
5 changes: 1 addition & 4 deletions pkg/sql/delete.go
Original file line number Diff line number Diff line change
Expand Up @@ -238,9 +238,6 @@ func (d *deleteNode) DebugValues() debugValues {
return d.run.rows.DebugValues()
}

func (d *deleteNode) Ordering() orderingInfo {
return d.run.rows.Ordering()
}
func (d *deleteNode) Ordering() orderingInfo { return orderingInfo{} }

func (d *deleteNode) SetLimitHint(numRows int64, soft bool) {}
func (d *deleteNode) setNeededColumns(_ []bool) {}
2 changes: 0 additions & 2 deletions pkg/sql/distinct.go
Original file line number Diff line number Diff line change
Expand Up @@ -244,8 +244,6 @@ func (n *distinctNode) SetLimitHint(numRows int64, soft bool) {
n.plan.SetLimitHint(numRows, true)
}

func (*distinctNode) setNeededColumns(_ []bool) {}

func (n *distinctNode) Close() {
n.plan.Close()
n.prefixSeen = nil
Expand Down
4 changes: 0 additions & 4 deletions pkg/sql/drop.go
Original file line number Diff line number Diff line change
Expand Up @@ -217,7 +217,6 @@ func (n *dropDatabaseNode) Ordering() orderingInfo { return orderingInfo{}
func (n *dropDatabaseNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *dropDatabaseNode) DebugValues() debugValues { return debugValues{} }
func (n *dropDatabaseNode) SetLimitHint(_ int64, _ bool) {}
func (n *dropDatabaseNode) setNeededColumns(_ []bool) {}
func (n *dropDatabaseNode) MarkDebug(mode explainMode) {}

type dropIndexNode struct {
Expand Down Expand Up @@ -394,7 +393,6 @@ func (n *dropIndexNode) Ordering() orderingInfo { return orderingInfo{} }
func (n *dropIndexNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *dropIndexNode) DebugValues() debugValues { return debugValues{} }
func (n *dropIndexNode) SetLimitHint(_ int64, _ bool) {}
func (n *dropIndexNode) setNeededColumns(_ []bool) {}
func (n *dropIndexNode) MarkDebug(mode explainMode) {}

type dropViewNode struct {
Expand Down Expand Up @@ -507,7 +505,6 @@ func (n *dropViewNode) Ordering() orderingInfo { return orderingInfo{} }
func (n *dropViewNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *dropViewNode) DebugValues() debugValues { return debugValues{} }
func (n *dropViewNode) SetLimitHint(_ int64, _ bool) {}
func (n *dropViewNode) setNeededColumns(_ []bool) {}
func (n *dropViewNode) MarkDebug(mode explainMode) {}

type dropTableNode struct {
Expand Down Expand Up @@ -732,7 +729,6 @@ func (n *dropTableNode) Ordering() orderingInfo { return orderingInfo{} }
func (n *dropTableNode) Values() parser.DTuple { return parser.DTuple{} }
func (n *dropTableNode) DebugValues() debugValues { return debugValues{} }
func (n *dropTableNode) SetLimitHint(_ int64, _ bool) {}
func (n *dropTableNode) setNeededColumns(_ []bool) {}
func (n *dropTableNode) MarkDebug(mode explainMode) {}

// dropTableOrViewPrepare/dropTableImpl is used to drop a single table by
Expand Down
1 change: 0 additions & 1 deletion pkg/sql/empty.go
Original file line number Diff line number Diff line change
Expand Up @@ -32,7 +32,6 @@ func (*emptyNode) Ordering() orderingInfo { return orderingInfo{} }
func (*emptyNode) Values() parser.DTuple { return nil }
func (*emptyNode) Start() error { return nil }
func (*emptyNode) SetLimitHint(_ int64, _ bool) {}
func (*emptyNode) setNeededColumns(_ []bool) {}
func (*emptyNode) MarkDebug(_ explainMode) {}
func (*emptyNode) expandPlan() error { return nil }
func (*emptyNode) Close() {}
Expand Down
6 changes: 4 additions & 2 deletions pkg/sql/explain.go
Original file line number Diff line number Diff line change
Expand Up @@ -44,6 +44,7 @@ var explainStrings = []string{"", "debug", "plan", "trace", "types"}
func (p *planner) Explain(n *parser.Explain, autoCommit bool) (planNode, error) {
mode := explainNone

optimized := true
expanded := true
normalizeExprs := true
explainer := explainer{
Expand Down Expand Up @@ -92,6 +93,8 @@ func (p *planner) Explain(n *parser.Explain, autoCommit bool) (planNode, error)
expanded = false
} else if strings.EqualFold(opt, "NONORMALIZE") {
normalizeExprs = false
} else if strings.EqualFold(opt, "NOOPTIMIZE") {
optimized = false
} else {
return nil, fmt.Errorf("unsupported EXPLAIN option: %s", opt)
}
Expand Down Expand Up @@ -130,7 +133,7 @@ func (p *planner) Explain(n *parser.Explain, autoCommit bool) (planNode, error)
// We may want to show placeholder types, so ensure no values
// are missing.
p.semaCtx.Placeholders.FillUnassigned()
return p.makeExplainPlanNode(explainer, expanded, plan), nil
return p.makeExplainPlanNode(explainer, expanded, optimized, plan), nil

case explainTrace:
return p.makeTraceNode(plan, p.txn), nil
Expand Down Expand Up @@ -258,4 +261,3 @@ func (n *explainDebugNode) Values() parser.DTuple {
func (*explainDebugNode) MarkDebug(_ explainMode) {}
func (*explainDebugNode) DebugValues() debugValues { return debugValues{} }
func (*explainDebugNode) SetLimitHint(_ int64, _ bool) {}
func (*explainDebugNode) setNeededColumns(_ []bool) {}
31 changes: 26 additions & 5 deletions pkg/sql/explain_plan.go
Original file line number Diff line number Diff line change
Expand Up @@ -68,7 +68,9 @@ type explainer struct {
}

// newExplainPlanNode instantiates a planNode that runs an EXPLAIN query.
func (p *planner) makeExplainPlanNode(explainer explainer, expanded bool, plan planNode) planNode {
func (p *planner) makeExplainPlanNode(
explainer explainer, expanded, optimized bool, plan planNode,
) planNode {
columns := ResultColumns{
// Level is the depth of the node in the tree.
{Name: "Level", Typ: parser.TypeInt},
Expand All @@ -95,6 +97,7 @@ func (p *planner) makeExplainPlanNode(explainer explainer, expanded bool, plan p
p: p,
explainer: explainer,
expanded: expanded,
optimized: optimized,
plan: plan,
results: p.newContainerValuesNode(columns, 0),
}
Expand Down Expand Up @@ -258,9 +261,18 @@ func formatColumns(cols ResultColumns, printTypes bool) string {
type explainPlanNode struct {
p *planner
explainer explainer
expanded bool
plan planNode
results *valuesNode

// plan is the sub-node being explained.
plan planNode

// results is the container for EXPLAIN's output.
results *valuesNode

// expanded indicates whether to invoke expandPlan() on the sub-node.
expanded bool

// optimized indicates whether to invoke initNeededColumns() on the sub-node.
optimized bool
}

func (e *explainPlanNode) Next() (bool, error) { return e.results.Next() }
Expand All @@ -269,10 +281,14 @@ func (e *explainPlanNode) Ordering() orderingInfo { return e.results.Order
func (e *explainPlanNode) Values() parser.DTuple { return e.results.Values() }
func (e *explainPlanNode) DebugValues() debugValues { return debugValues{} }
func (e *explainPlanNode) SetLimitHint(n int64, s bool) { e.results.SetLimitHint(n, s) }
func (e *explainPlanNode) setNeededColumns(_ []bool) {}
func (e *explainPlanNode) MarkDebug(mode explainMode) {}
func (e *explainPlanNode) expandPlan() error {

if e.expanded {
if e.optimized {
e.p.initNeededColumns(e.plan, allColumns(e.plan), false)
}

if err := e.plan.expandPlan(); err != nil {
return err
}
Expand All @@ -282,6 +298,11 @@ func (e *explainPlanNode) expandPlan() error {
// interested in.
e.plan.SetLimitHint(math.MaxInt64, true)
}

if e.optimized {
e.p.initNeededColumns(e.plan, allColumns(e.plan), true)
}

return nil
}

Expand Down
3 changes: 3 additions & 0 deletions pkg/sql/expr_filter.go
Original file line number Diff line number Diff line change
Expand Up @@ -108,6 +108,9 @@ func exprCheckVars(expr parser.Expr, conv varConvertFunc) bool {
// Convert the variables in the given expression; the expression must only contain
// variables known to the conversion function (exprCheckVars should be used first).
func exprConvertVars(expr parser.TypedExpr, conv varConvertFunc) parser.TypedExpr {
if expr == nil {
return expr
}
v := varConvertVisitor{justCheck: false, conv: conv}
ret, _ := parser.WalkExpr(&v, expr)
return ret.(parser.TypedExpr)
Expand Down
1 change: 0 additions & 1 deletion pkg/sql/generator.go
Original file line number Diff line number Diff line change
Expand Up @@ -136,4 +136,3 @@ func (n *valueGenerator) Values() parser.DTuple { return n.gen.Values() }
func (n *valueGenerator) MarkDebug(_ explainMode) {}
func (n *valueGenerator) Columns() ResultColumns { return n.columns }
func (n *valueGenerator) SetLimitHint(_ int64, _ bool) {}
func (n *valueGenerator) setNeededColumns(_ []bool) {}
1 change: 0 additions & 1 deletion pkg/sql/group.go
Original file line number Diff line number Diff line change
Expand Up @@ -414,7 +414,6 @@ func (n *groupNode) computeAggregates() error {
}

func (*groupNode) SetLimitHint(_ int64, _ bool) {}
func (*groupNode) setNeededColumns(_ []bool) {}

func (n *groupNode) Close() {
n.plan.Close()
Expand Down
36 changes: 17 additions & 19 deletions pkg/sql/index_join.go
Original file line number Diff line number Diff line change
Expand Up @@ -37,6 +37,7 @@ type indexJoinNode struct {
table *scanNode
primaryKeyPrefix roachpb.Key
colIDtoRowIndex map[sqlbase.ColumnID]int
valNeededIndex []bool
explain explainMode
debugVals debugValues
}
Expand Down Expand Up @@ -77,49 +78,48 @@ func (p *planner) makeIndexJoin(
colIDtoRowIndex[colID] = idx
}

// Transfer needed columns set to the table node.
table.setNeededColumns(origScan.valNeededForCol)

// For the index node, we need values for columns that are part of the index.
// TODO(radu): we could reduce this further - we only need the PK columns plus
// whatever filters may be used by the filter below.
valNeededIndex := make([]bool, len(origScan.valNeededForCol))
for _, idx := range colIDtoRowIndex {
valNeededIndex[idx] = true
}
indexScan.setNeededColumns(valNeededIndex)

if origScan.filter != nil {
// Transfer the filter to the table node. We must first convert the
// IndexedVars associated with indexNode.
convFunc := func(expr parser.VariableExpr) (ok bool, newExpr parser.VariableExpr) {
iv := expr.(*parser.IndexedVar)
return true, table.filterVars.IndexedVar(iv.Idx)
}
table.filter = exprConvertVars(origScan.filter, convFunc)
// Now we split the filter by extracting the part that can be
// evaluated using just the index columns.

// Now we split the filter by extracting the part that can be evaluated using just the index
// columns.
// Since we are re-populating the IndexedVars, reset the helper
// first so that the new vars are properly accounted for.
indexScan.filterVars.Reset()
splitFunc := func(expr parser.VariableExpr) (ok bool, newExpr parser.VariableExpr) {
colIdx := expr.(*parser.IndexedVar).Idx
if !indexScan.valNeededForCol[colIdx] {
if !valNeededIndex[colIdx] {
return false, nil
}
return true, indexScan.filterVars.IndexedVar(colIdx)
}
indexScan.filter, table.filter = splitFilter(table.filter, splitFunc)
indexScan.filter, table.filter = splitFilter(origScan.filter, splitFunc)
}

// Ensure that the indexed vars are transferred to the scanNodes fully.
table.filterVars.Reset()
table.filter = table.filterVars.Rebind(table.filter)

indexScan.initOrdering(exactPrefix)

primaryKeyPrefix := roachpb.Key(sqlbase.MakeIndexKeyPrefix(&table.desc, table.index.ID))

return &indexJoinNode{
node := &indexJoinNode{
index: indexScan,
table: table,
primaryKeyPrefix: primaryKeyPrefix,
colIDtoRowIndex: colIDtoRowIndex,
}, indexScan
valNeededIndex: valNeededIndex,
}

return node, indexScan
}

func (n *indexJoinNode) Columns() ResultColumns {
Expand Down Expand Up @@ -243,8 +243,6 @@ func (n *indexJoinNode) SetLimitHint(numRows int64, soft bool) {
n.index.SetLimitHint(numRows, soft)
}

func (*indexJoinNode) setNeededColumns(_ []bool) {}

func (n *indexJoinNode) Close() {
n.index.Close()
n.table.Close()
Expand Down
5 changes: 1 addition & 4 deletions pkg/sql/insert.go
Original file line number Diff line number Diff line change
Expand Up @@ -575,9 +575,6 @@ func (n *insertNode) DebugValues() debugValues {
return n.run.rows.DebugValues()
}

func (n *insertNode) Ordering() orderingInfo {
return n.run.rows.Ordering()
}
func (n *insertNode) Ordering() orderingInfo { return orderingInfo{} }

func (n *insertNode) SetLimitHint(numRows int64, soft bool) {}
func (n *insertNode) setNeededColumns(_ []bool) {}
10 changes: 0 additions & 10 deletions pkg/sql/join.go
Original file line number Diff line number Diff line change
Expand Up @@ -273,16 +273,6 @@ func (p *planner) makeJoin(
// SetLimitHint implements the planNode interface.
func (n *joinNode) SetLimitHint(numRows int64, soft bool) {}

// setNeededColumns implements the planNode interface.
func (n *joinNode) setNeededColumns(needed []bool) {
leftNeeded, rightNeeded := n.pred.getNeededColumns(needed)
n.left.plan.setNeededColumns(leftNeeded)
n.right.plan.setNeededColumns(rightNeeded)
for i, v := range needed {
n.columns[i].omitted = !v
}
}

// expandPlan implements the planNode interface.
func (n *joinNode) expandPlan() error {
if err := n.planner.expandSubqueryPlans(n.pred.filter); err != nil {
Expand Down
9 changes: 8 additions & 1 deletion pkg/sql/join_predicate.go
Original file line number Diff line number Diff line change
Expand Up @@ -399,8 +399,15 @@ func (p *joinPredicate) eval(
return true, nil
}

// getNeededColumns figures out the columns needed for the two sources.
// getNeededColumns figures out the columns needed for the two
// sources. This takes into account both the equality columns and the
// predicate expression.
func (p *joinPredicate) getNeededColumns(neededJoined []bool) ([]bool, []bool) {
// Reset the helper and rebind the variable to detect which columns
// are effectively needed.
p.iVarHelper.Reset()
p.filter = p.iVarHelper.Rebind(p.filter)

// The columns that are part of the expression are always needed.
neededJoined = append([]bool(nil), neededJoined...)
for i := range neededJoined {
Expand Down
2 changes: 0 additions & 2 deletions pkg/sql/limit.go
Original file line number Diff line number Diff line change
Expand Up @@ -289,5 +289,3 @@ func (n *limitNode) SetLimitHint(count int64, soft bool) {
}
n.plan.SetLimitHint(getLimit(hintCount, n.offset), soft)
}

func (*limitNode) setNeededColumns(_ []bool) {}
Loading

0 comments on commit 789f749

Please sign in to comment.