Skip to content

Commit

Permalink
planner,expression: use constraint propagation in partition pruning (#…
Browse files Browse the repository at this point in the history
  • Loading branch information
tiancaiamao authored Jan 17, 2019
1 parent 59c7b69 commit b339c02
Show file tree
Hide file tree
Showing 8 changed files with 1,575 additions and 2,504 deletions.
3,924 changes: 1,446 additions & 2,478 deletions cmd/explaintest/r/partition_pruning.result

Large diffs are not rendered by default.

21 changes: 20 additions & 1 deletion expression/constraint_propagation.go
Original file line number Diff line number Diff line change
Expand Up @@ -249,7 +249,21 @@ func ruleColumnOPConst(ctx sessionctx.Context, i, j int, exprs *exprSet) {
return
}
}
if !col1.Equal(ctx, col2) {

// Make sure col1 and col2 are the same column.
// Can't use col1.Equal(ctx, col2) here, because they are not generated in one
// expression and their UniqueID are not the same.
if col1.ColName.L != col2.ColName.L {
return
}
if col1.OrigColName.L != "" &&
col2.OrigColName.L != "" &&
col1.OrigColName.L != col2.OrigColName.L {
return
}
if col1.OrigTblName.L != "" &&
col2.OrigTblName.L != "" &&
col1.OrigColName.L != col2.OrigColName.L {
return
}
v, isNull, err := compareConstant(ctx, negOP(OP2), fc1, con2)
Expand Down Expand Up @@ -300,3 +314,8 @@ func compareConstant(ctx sessionctx.Context, fn string, c1, c2 Expression) (int6
}
return cmp.EvalInt(ctx, chunk.Row{})
}

// NewPartitionPruneSolver returns a constraintSolver for partition pruning.
func NewPartitionPruneSolver() constraintSolver {
return newConstraintSolver(ruleColumnOPConst)
}
1 change: 0 additions & 1 deletion go.mod
Original file line number Diff line number Diff line change
Expand Up @@ -81,7 +81,6 @@ require (
golang.org/x/sys v0.0.0-20190109145017-48ac38b7c8cb // indirect
golang.org/x/text v0.3.0
golang.org/x/time v0.0.0-20181108054448-85acf8d2951c // indirect
golang.org/x/tools v0.0.0-20190110015856-aa033095749b // indirect
google.golang.org/genproto v0.0.0-20190108161440-ae2f86662275 // indirect
google.golang.org/grpc v1.17.0
gopkg.in/natefinch/lumberjack.v2 v2.0.0
Expand Down
2 changes: 0 additions & 2 deletions go.sum
Original file line number Diff line number Diff line change
Expand Up @@ -200,8 +200,6 @@ golang.org/x/time v0.0.0-20181108054448-85acf8d2951c h1:fqgJT0MGcGpPgpWU7VRdRjuA
golang.org/x/time v0.0.0-20181108054448-85acf8d2951c/go.mod h1:tRJNPiyCQ0inRvYxbN9jk5I+vvW/OXSQhTDSoE431IQ=
golang.org/x/tools v0.0.0-20180828015842-6cd1fcedba52 h1:JG/0uqcGdTNgq7FdU+61l5Pdmb8putNZlXb65bJBROs=
golang.org/x/tools v0.0.0-20180828015842-6cd1fcedba52/go.mod h1:n7NCudcB/nEzxVGmLbDWY5pfWTLqBcC2KZ6jyYvM4mQ=
golang.org/x/tools v0.0.0-20190110015856-aa033095749b h1:G5tsw1T5VA7PD7VmXyGtX/hQp3ABPSCPRKVfsdUcVxs=
golang.org/x/tools v0.0.0-20190110015856-aa033095749b/go.mod h1:n7NCudcB/nEzxVGmLbDWY5pfWTLqBcC2KZ6jyYvM4mQ=
google.golang.org/appengine v1.1.0/go.mod h1:EbEs0AVv82hx2wNQdGPgUI5lhzA/G0D9YwlJXL52JkM=
google.golang.org/genproto v0.0.0-20180817151627-c66870c02cf8 h1:Nw54tB0rB7hY/N0NQvRW8DG4Yk3Q6T9cu9RcFQDu1tc=
google.golang.org/genproto v0.0.0-20180817151627-c66870c02cf8/go.mod h1:JiN7NxoALGmiZfu7CAH4rXhgtRTLTxftemlI0sWmxmc=
Expand Down
3 changes: 3 additions & 0 deletions planner/core/logical_plans.go
Original file line number Diff line number Diff line change
Expand Up @@ -300,6 +300,9 @@ type DataSource struct {

// pushedDownConds are the conditions that will be pushed down to coprocessor.
pushedDownConds []expression.Expression
// allConds contains all the filters on this table. For now it's maintained
// in predicate push down and used only in partition pruning.
allConds []expression.Expression

// relevantIndices means the indices match the push down conditions
relevantIndices []bool
Expand Down
63 changes: 63 additions & 0 deletions planner/core/partition_pruning_test.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,63 @@
// Copyright 2018 PingCAP, Inc.
//
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
// // Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// See the License for the specific language governing permissions and
// limitations under the License.

package core

import (
. "github.com/pingcap/check"
"github.com/pingcap/parser"
"github.com/pingcap/parser/ast"
"github.com/pingcap/parser/model"
"github.com/pingcap/tidb/ddl"
"github.com/pingcap/tidb/expression"
"github.com/pingcap/tidb/util/mock"
)

var _ = Suite(&testPartitionPruningSuite{})

type testPartitionPruningSuite struct {
partitionProcessor
}

func (s *testPartitionPruningSuite) TestCanBePrune(c *C) {
p := parser.New()
stmt, err := p.ParseOneStmt("create table t (d datetime not null)", "", "")
c.Assert(err, IsNil)
tblInfo, err := ddl.BuildTableInfoFromAST(stmt.(*ast.CreateTableStmt))
c.Assert(err, IsNil)

// For the following case:
// CREATE TABLE t1 ( recdate DATETIME NOT NULL )
// PARTITION BY RANGE( TO_DAYS(recdate) ) (
// PARTITION p0 VALUES LESS THAN ( TO_DAYS('2007-03-08') ),
// PARTITION p1 VALUES LESS THAN ( TO_DAYS('2007-04-01') )
// );
// SELECT * FROM t1 WHERE recdate < '2007-03-08 00:00:00';
// SELECT * FROM t1 WHERE recdate > '2018-03-08 00:00:00';

ctx := mock.NewContext()
columns := expression.ColumnInfos2ColumnsWithDBName(ctx, model.NewCIStr("t"), tblInfo.Name, tblInfo.Columns)
schema := expression.NewSchema(columns...)
partitionExpr, err := expression.ParseSimpleExprsWithSchema(ctx, "to_days(d) < to_days('2007-03-08') and to_days(d) >= to_days('2007-03-07')", schema)
c.Assert(err, IsNil)
queryExpr, err := expression.ParseSimpleExprsWithSchema(ctx, "d < '2000-03-08 00:00:00'", schema)
c.Assert(err, IsNil)
succ, err := s.canBePruned(ctx, nil, partitionExpr[0], queryExpr)
c.Assert(err, IsNil)
c.Assert(succ, IsTrue)

queryExpr, err = expression.ParseSimpleExprsWithSchema(ctx, "d > '2018-03-08 00:00:00'", schema)
c.Assert(err, IsNil)
succ, err = s.canBePruned(ctx, nil, partitionExpr[0], queryExpr)
c.Assert(err, IsNil)
c.Assert(succ, IsTrue)
}
64 changes: 42 additions & 22 deletions planner/core/rule_partition_processor.go
Original file line number Diff line number Diff line change
Expand Up @@ -18,6 +18,7 @@ import (
"github.com/pingcap/tidb/expression"
"github.com/pingcap/tidb/sessionctx"
"github.com/pingcap/tidb/table/tables"
"github.com/pingcap/tidb/util/chunk"
"github.com/pingcap/tidb/util/ranger"
)

Expand All @@ -38,9 +39,6 @@ import (
type partitionProcessor struct{}

func (s *partitionProcessor) optimize(lp LogicalPlan) (LogicalPlan, error) {
// NOTE: partitionProcessor will assume all filter conditions are pushed down to
// DataSource, there will not be a Selection->DataSource case, so the rewrite just
// handle the DataSource node.
return s.rewriteDataSource(lp)
}

Expand Down Expand Up @@ -89,15 +87,13 @@ func (s *partitionProcessor) prune(ds *DataSource) (LogicalPlan, error) {
// partitions according to the filter conditions pushed to the DataSource.
children := make([]LogicalPlan, 0, len(pi.Definitions))
for i, expr := range partitionExprs {
if col != nil {
// If the selection condition would never be satisified, prune that partition.
prune, err := s.canBePrune(ds.context(), col, expr, ds.pushedDownConds)
if err != nil {
return nil, errors.Trace(err)
}
if prune {
continue
}
// If the select condition would never be satisified, prune that partition.
pruned, err := s.canBePruned(ds.context(), col, expr, ds.allConds)
if err != nil {
return nil, errors.Trace(err)
}
if pruned {
continue
}
// This is for `table partition (p0,p1)` syntax, only union the specified partition if has specified partitions.
if len(ds.partitionNames) != 0 {
Expand Down Expand Up @@ -134,17 +130,41 @@ func (s *partitionProcessor) prune(ds *DataSource) (LogicalPlan, error) {
return unionAll, nil
}

// canBePrune checks if partition expression will never meets the selection condition.
var solver = expression.NewPartitionPruneSolver()

// canBePruned checks if partition expression will never meets the selection condition.
// For example, partition by column a > 3, and select condition is a < 3, then canBePrune returns true.
func (s *partitionProcessor) canBePrune(ctx sessionctx.Context, col *expression.Column, partitionCond expression.Expression, copConds []expression.Expression) (bool, error) {
conds := make([]expression.Expression, 0, 1+len(copConds))
conds = append(conds, partitionCond)
conds = append(conds, copConds...)
conds = expression.PropagateConstant(ctx, conds)

// Calculate the column range to prune.
accessConds := ranger.ExtractAccessConditionsForColumn(conds, col.UniqueID)
r, err := ranger.BuildColumnRange(accessConds, ctx.GetSessionVars().StmtCtx, col.RetType)
func (s *partitionProcessor) canBePruned(sctx sessionctx.Context, partCol *expression.Column, partExpr expression.Expression, filterExprs []expression.Expression) (bool, error) {
conds := make([]expression.Expression, 0, 1+len(filterExprs))
conds = append(conds, partExpr)
conds = append(conds, filterExprs...)
conds = expression.PropagateConstant(sctx, conds)
conds = solver.Solve(sctx, conds)

if len(conds) == 1 {
// Constant false.
if con, ok := conds[0].(*expression.Constant); ok && con.DeferredExpr == nil {
ret, err := expression.EvalBool(sctx, expression.CNFExprs{con}, chunk.Row{})
if err == nil && ret == false {
return true, nil
}
}
// Not a constant false, but this is the only condition, it can't be pruned.
return false, nil
}

// Calculates the column range to prune.
if partCol == nil {
// If partition column is nil, we can't calculate range, so we can't prune
// partition by range.
return false, nil
}

// TODO: Remove prune by calculating range. Current constraint propagate doesn't
// handle the null condition, while calculate range can prune something like:
// "select * from t where t is null"
accessConds := ranger.ExtractAccessConditionsForColumn(conds, partCol.UniqueID)
r, err := ranger.BuildColumnRange(accessConds, sctx.GetSessionVars().StmtCtx, partCol.RetType)
if err != nil {
return false, errors.Trace(err)
}
Expand Down
1 change: 1 addition & 0 deletions planner/core/rule_predicate_push_down.go
Original file line number Diff line number Diff line change
Expand Up @@ -97,6 +97,7 @@ func (p *LogicalUnionScan) PredicatePushDown(predicates []expression.Expression)

// PredicatePushDown implements LogicalPlan PredicatePushDown interface.
func (ds *DataSource) PredicatePushDown(predicates []expression.Expression) ([]expression.Expression, LogicalPlan) {
ds.allConds = predicates
_, ds.pushedDownConds, predicates = expression.ExpressionsToPB(ds.ctx.GetSessionVars().StmtCtx, predicates, ds.ctx.GetClient())
return predicates, ds
}
Expand Down

0 comments on commit b339c02

Please sign in to comment.