Skip to content

Commit

Permalink
planner: add HashCode method for LogicalPlan & use PlanHash for Group…
Browse files Browse the repository at this point in the history
…Expr's fingerPrint (pingcap#14224)
  • Loading branch information
francis0407 authored and sre-bot committed Jan 10, 2020
1 parent b6aca51 commit 1b34cc2
Show file tree
Hide file tree
Showing 7 changed files with 252 additions and 52 deletions.
Original file line number Diff line number Diff line change
Expand Up @@ -1065,7 +1065,6 @@
"SQL": "select a from (select floor(a) as a from t) as t2",
"Result": [
"Group#0 Schema:[Column#13]",
" Projection_4 input:[Group#1], floor(test.t.a)->Column#13",
" Projection_2 input:[Group#1], floor(test.t.a)->Column#13",
"Group#1 Schema:[test.t.a]",
" TableScan_1 table:t"
Expand Down
99 changes: 99 additions & 0 deletions planner/core/hashcode.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,99 @@
// Copyright 2019 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 (
"bytes"
"encoding/binary"
"sort"

"github.com/pingcap/tidb/util/plancodec"
)

func encodeIntAsUint32(result []byte, value int) []byte {
var buf [4]byte
binary.BigEndian.PutUint32(buf[:], uint32(value))
return append(result, buf[:]...)
}

// HashCode implements LogicalPlan interface.
func (p *baseLogicalPlan) HashCode() []byte {
// We use PlanID for the default hash, so if two plans do not have
// the same id, the hash value will never be the same.
result := make([]byte, 0, 4)
result = encodeIntAsUint32(result, p.id)
return result
}

// HashCode implements LogicalPlan interface.
func (p *LogicalProjection) HashCode() []byte {
// PlanType + SelectOffset + ExprNum + [Exprs]
// Expressions are commonly `Column`s, whose hashcode has the length 9, so
// we pre-alloc 10 bytes for each expr's hashcode.
result := make([]byte, 0, 12+len(p.Exprs)*10)
result = encodeIntAsUint32(result, plancodec.TypeStringToPhysicalID(p.tp))
result = encodeIntAsUint32(result, p.SelectBlockOffset())
result = encodeIntAsUint32(result, len(p.Exprs))
for _, expr := range p.Exprs {
exprHashCode := expr.HashCode(p.ctx.GetSessionVars().StmtCtx)
result = encodeIntAsUint32(result, len(exprHashCode))
result = append(result, exprHashCode...)
}
return result
}

// HashCode implements LogicalPlan interface.
func (p *LogicalTableDual) HashCode() []byte {
// PlanType + SelectOffset + RowCount
result := make([]byte, 0, 12)
result = encodeIntAsUint32(result, plancodec.TypeStringToPhysicalID(p.tp))
result = encodeIntAsUint32(result, p.SelectBlockOffset())
result = encodeIntAsUint32(result, p.RowCount)
return result
}

// HashCode implements LogicalPlan interface.
func (p *LogicalSelection) HashCode() []byte {
// PlanType + SelectOffset + ConditionNum + [Conditions]
// Conditions are commonly `ScalarFunction`s, whose hashcode usually has a
// length larger than 20, so we pre-alloc 25 bytes for each expr's hashcode.
result := make([]byte, 0, 12+len(p.Conditions)*25)
result = encodeIntAsUint32(result, plancodec.TypeStringToPhysicalID(p.tp))
result = encodeIntAsUint32(result, p.SelectBlockOffset())
result = encodeIntAsUint32(result, len(p.Conditions))

condHashCodes := make([][]byte, len(p.Conditions))
for i, expr := range p.Conditions {
condHashCodes[i] = expr.HashCode(p.ctx.GetSessionVars().StmtCtx)
}
// Sort the conditions, so `a > 1 and a < 100` can equal to `a < 100 and a > 1`.
sort.Slice(condHashCodes, func(i, j int) bool { return bytes.Compare(condHashCodes[i], condHashCodes[j]) < 0 })

for _, condHashCode := range condHashCodes {
result = encodeIntAsUint32(result, len(condHashCode))
result = append(result, condHashCode...)
}
return result
}

// HashCode implements LogicalPlan interface.
func (p *LogicalLimit) HashCode() []byte {
// PlanType + SelectOffset + Offset + Count
result := make([]byte, 24)
binary.BigEndian.PutUint32(result, uint32(plancodec.TypeStringToPhysicalID(p.tp)))
binary.BigEndian.PutUint32(result[4:], uint32(p.SelectBlockOffset()))
binary.BigEndian.PutUint64(result[8:], p.Offset)
binary.BigEndian.PutUint64(result[16:], p.Count)
return result
}
4 changes: 4 additions & 0 deletions planner/core/plan.go
Original file line number Diff line number Diff line change
Expand Up @@ -80,6 +80,10 @@ func enforceProperty(p *property.PhysicalProperty, tsk task, ctx sessionctx.Cont
type LogicalPlan interface {
Plan

// HashCode encodes a LogicalPlan to fast compare whether a LogicalPlan equals to another.
// We use a strict encode method here which ensures there is no conflict.
HashCode() []byte

// PredicatePushDown pushes down the predicates in the where/on/having clauses as deeply as possible.
// It will accept a predicate that is an expression slice, and return the expressions that can't be pushed.
// Because it might change the root if the having clause exists, we need to return a plan that represents a new root.
Expand Down
112 changes: 72 additions & 40 deletions planner/memo/expr_iterator_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,6 +15,7 @@ package memo

import (
. "github.com/pingcap/check"
"github.com/pingcap/tidb/expression"
plannercore "github.com/pingcap/tidb/planner/core"
)

Expand Down Expand Up @@ -58,17 +59,27 @@ func (s *testMemoSuite) TestNewExprIterFromGroupElem(c *C) {
}

func (s *testMemoSuite) TestExprIterNext(c *C) {
g0 := NewGroupWithSchema(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)), s.schema)
g0.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)))

g1 := NewGroupWithSchema(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)), s.schema)
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)))
g0 := NewGroupWithSchema(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.Zero}}.Init(s.sctx, 0)), s.schema)
g0.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 1}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.One}}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 2}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.Null}}.Init(s.sctx, 0)))

g1 := NewGroupWithSchema(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.Null}}.Init(s.sctx, 0)), s.schema)
g1.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 3}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.One}}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 4}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.Zero}}.Init(s.sctx, 0)))

expr := NewGroupExpr(plannercore.LogicalJoin{}.Init(s.sctx, 0))
expr.Children = append(expr.Children, g0)
Expand Down Expand Up @@ -102,26 +113,36 @@ func (s *testMemoSuite) TestExprIterNext(c *C) {
}

func (s *testMemoSuite) TestExprIterReset(c *C) {
g0 := NewGroupWithSchema(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)), s.schema)
g0.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)))

sel1 := NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0))
sel2 := NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0))
sel3 := NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0))
g0 := NewGroupWithSchema(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.Zero}}.Init(s.sctx, 0)), s.schema)
g0.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 1}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.One}}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 2}.Init(s.sctx, 0)))
g0.Insert(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.Null}}.Init(s.sctx, 0)))

sel1 := NewGroupExpr(plannercore.LogicalSelection{Conditions: []expression.Expression{expression.Null}}.Init(s.sctx, 0))
sel2 := NewGroupExpr(plannercore.LogicalSelection{Conditions: []expression.Expression{expression.One}}.Init(s.sctx, 0))
sel3 := NewGroupExpr(plannercore.LogicalSelection{Conditions: []expression.Expression{expression.Zero}}.Init(s.sctx, 0))
g1 := NewGroupWithSchema(sel1, s.schema)
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{Count: 3}.Init(s.sctx, 0)))
g1.Insert(sel2)
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{Count: 4}.Init(s.sctx, 0)))
g1.Insert(sel3)

g2 := NewGroupWithSchema(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)), s.schema)
g2.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)))
g2 := NewGroupWithSchema(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.Null}}.Init(s.sctx, 0)), s.schema)
g2.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 3}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.One}}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 4}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.Zero}}.Init(s.sctx, 0)))

// link join with Group 0 and 1
expr := NewGroupExpr(plannercore.LogicalJoin{}.Init(s.sctx, 0))
Expand Down Expand Up @@ -185,25 +206,36 @@ func countMatchedIter(group *Group, pattern *Pattern) int {
}

func (s *testMemoSuite) TestExprIterWithEngineType(c *C) {
g1 := NewGroupWithSchema(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)), s.schema).SetEngineType(EngineTiFlash)
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))

g2 := NewGroupWithSchema(NewGroupExpr(plannercore.LogicalSelection{}.Init(s.sctx, 0)), s.schema).SetEngineType(EngineTiKV)
g2.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(plannercore.LogicalProjection{}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(plannercore.LogicalLimit{}.Init(s.sctx, 0)))

flashGather := NewGroupExpr(plannercore.TiKVSingleGather{}.Init(s.sctx, 0))
g1 := NewGroupWithSchema(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.One}}.Init(s.sctx, 0)), s.schema).SetEngineType(EngineTiFlash)
g1.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 1}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.One}}.Init(s.sctx, 0)))
g1.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 2}.Init(s.sctx, 0)))

g2 := NewGroupWithSchema(NewGroupExpr(
plannercore.LogicalSelection{Conditions: []expression.Expression{expression.One}}.Init(s.sctx, 0)), s.schema).SetEngineType(EngineTiKV)
g2.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 2}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(
plannercore.LogicalProjection{Exprs: []expression.Expression{expression.One}}.Init(s.sctx, 0)))
g2.Insert(NewGroupExpr(
plannercore.LogicalLimit{Count: 3}.Init(s.sctx, 0)))

flashGather := NewGroupExpr(
plannercore.TiKVSingleGather{}.Init(s.sctx, 0))
flashGather.Children = append(flashGather.Children, g1)
g3 := NewGroupWithSchema(flashGather, s.schema).SetEngineType(EngineTiDB)

tikvGather := NewGroupExpr(plannercore.TiKVSingleGather{}.Init(s.sctx, 0))
tikvGather := NewGroupExpr(
plannercore.TiKVSingleGather{}.Init(s.sctx, 0))
tikvGather.Children = append(tikvGather.Children, g2)
g3.Insert(tikvGather)

join := NewGroupExpr(plannercore.LogicalJoin{}.Init(s.sctx, 0))
join := NewGroupExpr(
plannercore.LogicalJoin{}.Init(s.sctx, 0))
join.Children = append(join.Children, g3, g3)
g4 := NewGroupWithSchema(join, s.schema).SetEngineType(EngineTiDB)

Expand Down
16 changes: 11 additions & 5 deletions planner/memo/group_expr.go
Original file line number Diff line number Diff line change
Expand Up @@ -14,7 +14,7 @@
package memo

import (
"fmt"
"encoding/binary"
"reflect"

"github.com/pingcap/tidb/expression"
Expand Down Expand Up @@ -51,11 +51,17 @@ func NewGroupExpr(node plannercore.LogicalPlan) *GroupExpr {

// FingerPrint gets the unique fingerprint of the Group expression.
func (e *GroupExpr) FingerPrint() string {
if e.selfFingerprint == "" {
e.selfFingerprint = fmt.Sprintf("%v", e.ExprNode.ID())
for i := range e.Children {
e.selfFingerprint += e.Children[i].FingerPrint()
if len(e.selfFingerprint) == 0 {
planHash := e.ExprNode.HashCode()
buffer := make([]byte, 2, 2+len(e.Children)*8+len(planHash))
binary.BigEndian.PutUint16(buffer, uint16(len(e.Children)))
for _, child := range e.Children {
var buf [8]byte
binary.BigEndian.PutUint64(buf[:], uint64(reflect.ValueOf(child).Pointer()))
buffer = append(buffer, buf[:]...)
}
buffer = append(buffer, planHash...)
e.selfFingerprint = string(buffer)
}
return e.selfFingerprint
}
Expand Down
18 changes: 14 additions & 4 deletions planner/memo/group_expr_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -14,7 +14,11 @@
package memo

import (
"encoding/binary"
"reflect"

. "github.com/pingcap/check"
"github.com/pingcap/tidb/expression"
plannercore "github.com/pingcap/tidb/planner/core"
)

Expand All @@ -27,9 +31,15 @@ func (s *testMemoSuite) TestNewGroupExpr(c *C) {
}

func (s *testMemoSuite) TestGroupExprFingerprint(c *C) {
p := &plannercore.LogicalLimit{}
p := &plannercore.LogicalLimit{Count: 3}
expr := NewGroupExpr(p)

// we haven't set the id of the created LogicalLimit, so the result is 0.
c.Assert(expr.FingerPrint(), Equals, "0")
childGroup := NewGroupWithSchema(nil, expression.NewSchema())
expr.SetChildren(childGroup)
// ChildNum(2 bytes) + ChildPointer(8 bytes) + LogicalLimit HashCode
planHash := p.HashCode()
buffer := make([]byte, 10+len(planHash))
binary.BigEndian.PutUint16(buffer, 1)
binary.BigEndian.PutUint64(buffer[2:], uint64(reflect.ValueOf(childGroup).Pointer()))
copy(buffer[10:], planHash)
c.Assert(expr.FingerPrint(), Equals, string(buffer))
}
Loading

0 comments on commit 1b34cc2

Please sign in to comment.