Skip to content

Commit

Permalink
update
Browse files Browse the repository at this point in the history
Signed-off-by: Weizhen Wang <[email protected]>
  • Loading branch information
hawkingrei committed Jan 26, 2025
1 parent 740d77b commit 0ac6d80
Show file tree
Hide file tree
Showing 22 changed files with 141 additions and 158 deletions.
2 changes: 1 addition & 1 deletion pkg/executor/prepared_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -55,7 +55,7 @@ func TestPreparedNullParam(t *testing.T) {
ps := []*util.ProcessInfo{tkProcess}
tk.Session().SetSessionManager(&testkit.MockSessionManager{PS: ps})
tk.MustQuery(fmt.Sprintf("explain for connection %d", tkProcess.ID)).Check(testkit.Rows(
"TableDual_5 0.00 root rows:0"))
"TableDual_6 0.00 root rows:0"))
}
}

Expand Down
7 changes: 0 additions & 7 deletions pkg/expression/constant_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -176,13 +176,6 @@ func TestConstantPropagation(t *testing.T) {
},
result: "eq(Column#0, Column#1), le(cast(Column#0, double BINARY), rand())",
},
{
solver: []PropagateConstantSolver{newPropConstSolver()},
conditions: []Expression{
newFunctionWithMockCtx(ast.NullEQ, newColumn(0)),
},
result: "",
},
}
for _, tt := range tests {
for _, solver := range tt.solver {
Expand Down
3 changes: 2 additions & 1 deletion pkg/planner/core/casetest/rule/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -4,6 +4,7 @@ go_test(
name = "rule_test",
timeout = "short",
srcs = [
"dual_test.go",
"main_test.go",
"rule_derive_topn_from_window_test.go",
"rule_inject_extra_projection_test.go",
Expand All @@ -13,7 +14,7 @@ go_test(
],
data = glob(["testdata/**"]),
flaky = True,
shard_count = 8,
shard_count = 9,
deps = [
"//pkg/domain",
"//pkg/expression",
Expand Down
47 changes: 47 additions & 0 deletions pkg/planner/core/casetest/rule/dual_test.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,47 @@
// Copyright 2023 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,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.

package rule

import (
"testing"

"github.com/pingcap/tidb/pkg/testkit"
)

func TestDual(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("CREATE TABLE t (id INT PRIMARY KEY AUTO_INCREMENT,d INT);")
tk.MustQuery("explain select a from (select d as a from t where d = 0) k where k.a = 5").Check(testkit.Rows(
"TableDual_8 0.00 root rows:0"))
tk.MustQuery("select a from (select d as a from t where d = 0) k where k.a = 5").Check(testkit.Rows())
tk.MustQuery("explain select a from (select 1+2 as a from t where d = 0) k where k.a = 5").Check(testkit.Rows(
"Projection_8 0.00 root 3->Column#3",
"└─TableDual_9 0.00 root rows:0"))
tk.MustQuery("select a from (select 1+2 as a from t where d = 0) k where k.a = 5").Check(testkit.Rows())
tk.MustQuery("explain select * from t where d != null;").Check(testkit.Rows(
"TableDual_6 0.00 root rows:0"))
tk.MustQuery("explain select * from t where d > null;").Check(testkit.Rows(
"TableDual_6 0.00 root rows:0"))
tk.MustQuery("explain select * from t where d >= null;").Check(testkit.Rows(
"TableDual_6 0.00 root rows:0"))
tk.MustQuery("explain select * from t where d < null;").Check(testkit.Rows(
"TableDual_6 0.00 root rows:0"))
tk.MustQuery("explain select * from t where d <= null;").Check(testkit.Rows(
"TableDual_6 0.00 root rows:0"))
tk.MustQuery("explain select * from t where d = null;").Check(testkit.Rows(
"TableDual_6 0.00 root rows:0"))
}
17 changes: 2 additions & 15 deletions pkg/planner/core/casetest/rule/testdata/outer2inner_out.json
Original file line number Diff line number Diff line change
Expand Up @@ -161,14 +161,7 @@
{
"SQL": "select * from t3 as t1 left join t3 as t2 on t1.c3 = t2.c3 where t2.b3 != NULL; -- self join",
"Plan": [
"Projection 0.00 root test.t3.a3, test.t3.b3, test.t3.c3, test.t3.a3, test.t3.b3, test.t3.c3",
"└─HashJoin 0.00 root inner join, equal:[eq(test.t3.c3, test.t3.c3)]",
" ├─TableReader(Build) 0.00 root data:Selection",
" │ └─Selection 0.00 cop[tikv] ne(test.t3.b3, NULL), not(isnull(test.t3.c3))",
" │ └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo",
" └─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t3.c3))",
" └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo"
"TableDual 0.00 root rows:0"
]
},
{
Expand Down Expand Up @@ -434,13 +427,7 @@
{
"SQL": "select * from t3 as t1 left join t3 as t2 on t1.c3 = t2.c3 where t1.b3 != NULL -- negative case with self join",
"Plan": [
"HashJoin 0.00 root left outer join, left side:TableReader, equal:[eq(test.t3.c3, test.t3.c3)]",
"├─TableReader(Build) 0.00 root data:Selection",
"│ └─Selection 0.00 cop[tikv] ne(test.t3.b3, NULL)",
"│ └─TableFullScan 10000.00 cop[tikv] table:t1 keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t3.c3))",
" └─TableFullScan 10000.00 cop[tikv] table:t2 keep order:false, stats:pseudo"
"TableDual 0.00 root rows:0"
]
},
{
Expand Down
1 change: 1 addition & 0 deletions pkg/planner/core/issuetest/main_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -30,6 +30,7 @@ func TestMain(m *testing.M) {
testsetup.SetupForCommonTest()
flag.Parse()
opts := []goleak.Option{
goleak.IgnoreTopFunction("go.opencensus.io/stats/view.(*worker).start"),
goleak.IgnoreTopFunction("github.com/golang/glog.(*fileSink).flushDaemon"),
goleak.IgnoreTopFunction("github.com/bazelbuild/rules_go/go/tools/bzltestutil.RegisterTimeoutHandler.func1"),
goleak.IgnoreTopFunction("github.com/lestrrat-go/httprc.runFetchWorker"),
Expand Down
9 changes: 0 additions & 9 deletions pkg/planner/core/issuetest/planner_issue_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -177,12 +177,3 @@ func TestIssue58476(t *testing.T) {
` ├─TableRangeScan(Build) 3333.33 cop[tikv] table:t3 range:(0,+inf], keep order:false, stats:pseudo`,
` └─TableRowIDScan(Probe) 9990.00 cop[tikv] table:t3 keep order:false, stats:pseudo`))
}

func TestABC(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test;")
tk.MustExec("create table t (c1 year(4), c2 int, key(c1));")
tk.MustExec("insert into t values(2001, 1);")
tk.MustQuery("explain format = 'brief' select t1.c1, t2.c1 from t as t1 inner join t as t2 on t1.c1 = t2.c1 where t1.c1 != NULL;")
}
1 change: 0 additions & 1 deletion pkg/planner/core/logical_plan_builder.go
Original file line number Diff line number Diff line change
Expand Up @@ -562,7 +562,6 @@ func (b *PlanBuilder) buildJoin(ctx context.Context, joinNode *ast.Join) (base.L
b.optFlag = b.optFlag | rule.FlagJoinReOrder
b.optFlag |= rule.FlagPredicateSimplification
b.optFlag |= rule.FlagConvertOuterToInnerJoin
b.optFlag |= rule.FlagEliminaterDual

leftPlan, err := b.buildResultSetNode(ctx, joinNode.Left, false)
if err != nil {
Expand Down
2 changes: 1 addition & 1 deletion pkg/planner/core/logical_plans_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -330,7 +330,7 @@ func TestAntiSemiJoinConstFalse(t *testing.T) {
}{
{
sql: "select a from t t1 where not exists (select a from t t2 where t1.a = t2.a and t2.b = 1 and t2.b = 2)",
best: "Join{DataScan(t1)->DataScan(t2)}(test.t.a,test.t.a)->Projection",
best: "Join{DataScan(t1)->Dual}(test.t.a,test.t.a)->Projection",
joinType: "anti semi join",
},
}
Expand Down
1 change: 1 addition & 0 deletions pkg/planner/core/operator/logicalop/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -4,6 +4,7 @@ go_library(
name = "logicalop",
srcs = [
"base_logical_plan.go",
"expression_util.go",
"hash64_equals_generated.go",
"logical_aggregation.go",
"logical_apply.go",
Expand Down
4 changes: 0 additions & 4 deletions pkg/planner/core/operator/logicalop/base_logical_plan.go
Original file line number Diff line number Diff line change
Expand Up @@ -200,10 +200,6 @@ func (*BaseLogicalPlan) ConstantPropagation(_ base.LogicalPlan, _ int, _ *optimi
return nil
}

func (*BaseLogicalPlan) IsDual() bool {
return false
}

// PullUpConstantPredicates implements the LogicalPlan.<9th> interface.
func (*BaseLogicalPlan) PullUpConstantPredicates() []expression.Expression {
// Only LogicalProjection and LogicalSelection can get constant predicates
Expand Down
66 changes: 66 additions & 0 deletions pkg/planner/core/operator/logicalop/expression_util.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,66 @@
// Copyright 2025 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,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.

package logicalop

import (
"github.com/pingcap/tidb/pkg/expression"
"github.com/pingcap/tidb/pkg/parser/ast"
"github.com/pingcap/tidb/pkg/planner/core/base"
)

// isConstFalse is used to check whether the expression is a constant false expression.
func isConstFalse(expr expression.Expression) bool {
if e, ok := expr.(*expression.ScalarFunction); ok {
switch e.FuncName.L {
case ast.LT, ast.LE, ast.GT, ast.GE, ast.EQ, ast.NE:
if constExpr, ok := e.GetArgs()[1].(*expression.Constant); ok && constExpr.Value.IsNull() && constExpr.DeferredExpr == nil {
return true
}
}
}
return false
}

// Conds2TableDual builds a LogicalTableDual if cond is constant false or null.
func Conds2TableDual(p base.LogicalPlan, conds []expression.Expression) base.LogicalPlan {
for _, cond := range conds {
if isConstFalse(cond) {
if expression.MaybeOverOptimized4PlanCache(p.SCtx().GetExprCtx(), conds) {
return nil
}
dual := LogicalTableDual{}.Init(p.SCtx(), p.QueryBlockOffset())
dual.SetSchema(p.Schema())
return dual
}
}
if len(conds) != 1 {
return nil
}

con, ok := conds[0].(*expression.Constant)
if !ok {
return nil
}
sc := p.SCtx().GetSessionVars().StmtCtx
if expression.MaybeOverOptimized4PlanCache(p.SCtx().GetExprCtx(), []expression.Expression{con}) {
return nil
}
if isTrue, err := con.Value.ToBool(sc.TypeCtxOrDefault()); (err == nil && isTrue == 0) || con.Value.IsNull() {
dual := LogicalTableDual{}.Init(p.SCtx(), p.QueryBlockOffset())
dual.SetSchema(p.Schema())
return dual
}
return nil
}
5 changes: 5 additions & 0 deletions pkg/planner/core/operator/logicalop/logical_datasource.go
Original file line number Diff line number Diff line change
Expand Up @@ -155,6 +155,11 @@ func (ds *DataSource) PredicatePushDown(predicates []expression.Expression, opt
// TODO: remove it to the place building logical plan
predicates = utilfuncp.AddPrefix4ShardIndexes(ds, ds.SCtx(), predicates)
ds.AllConds = predicates
dual := Conds2TableDual(ds, ds.AllConds)
if dual != nil {
AppendTableDualTraceStep(ds, dual, predicates, opt)
return nil, dual
}
ds.PushedDownConds, predicates = expression.PushDownExprs(util.GetPushDownCtx(ds.SCtx()), predicates, kv.UnSpecified)
appendDataSourcePredicatePushDownTraceStep(ds, opt)
return predicates, ds
Expand Down
26 changes: 0 additions & 26 deletions pkg/planner/core/operator/logicalop/logical_join.go
Original file line number Diff line number Diff line change
Expand Up @@ -485,11 +485,6 @@ func (p *LogicalJoin) ConstantPropagation(parentPlan base.LogicalPlan, currentCh
return addCandidateSelection(p, currentChildIdx, parentPlan, candidateConstantPredicates, opt)
}

func (p *LogicalJoin) IsDual() bool {
fmt.Println("here")
return false
}

// PullUpConstantPredicates inherits the BaseLogicalPlan.LogicalPlan.<9th> implementation.

// RecursiveDeriveStats inherits the BaseLogicalPlan.LogicalPlan.<10th> implementation.
Expand Down Expand Up @@ -1873,27 +1868,6 @@ func deriveNotNullExpr(ctx base.PlanContext, expr expression.Expression, schema
return nil
}

// Conds2TableDual builds a LogicalTableDual if cond is constant false or null.
func Conds2TableDual(p base.LogicalPlan, conds []expression.Expression) base.LogicalPlan {
if len(conds) != 1 {
return nil
}
con, ok := conds[0].(*expression.Constant)
if !ok {
return nil
}
sc := p.SCtx().GetSessionVars().StmtCtx
if expression.MaybeOverOptimized4PlanCache(p.SCtx().GetExprCtx(), []expression.Expression{con}) {
return nil
}
if isTrue, err := con.Value.ToBool(sc.TypeCtxOrDefault()); (err == nil && isTrue == 0) || con.Value.IsNull() {
dual := LogicalTableDual{}.Init(p.SCtx(), p.QueryBlockOffset())
dual.SetSchema(p.Schema())
return dual
}
return nil
}

// BuildLogicalJoinSchema builds the schema for join operator.
func BuildLogicalJoinSchema(joinType JoinType, join base.LogicalPlan) *expression.Schema {
leftSchema := join.Children()[0].Schema()
Expand Down
1 change: 0 additions & 1 deletion pkg/planner/core/optimizer.go
Original file line number Diff line number Diff line change
Expand Up @@ -107,7 +107,6 @@ var optRuleList = []base.LogicalOptRule{
&ColumnPruner{}, // column pruning again at last, note it will mess up the results of buildKeySolver
&PushDownSequenceSolver{},
&ResolveExpand{},
&DualJoinEliminator{},
}

// Interaction Rule List
Expand Down
1 change: 0 additions & 1 deletion pkg/planner/core/rule/logical_rules.go
Original file line number Diff line number Diff line change
Expand Up @@ -42,7 +42,6 @@ const (
FlagPruneColumnsAgain
FlagPushDownSequence
FlagResolveExpand
FlagEliminaterDual
)

func setPredicatePushDownFlag(u uint64) uint64 {
Expand Down
43 changes: 0 additions & 43 deletions pkg/planner/core/rule_dual_join_elimination.go

This file was deleted.

4 changes: 2 additions & 2 deletions pkg/planner/core/testdata/plan_suite_unexported_out.json
Original file line number Diff line number Diff line change
Expand Up @@ -67,8 +67,8 @@
"Join{DataScan(a)->DataScan(b)}(test.t.a,test.t.a)->Aggr(count(1))->Projection",
"DataScan(t)->Projection->Projection",
"DataScan(t)->Projection->Projection",
"DataScan(t)->Projection->Projection",
"DataScan(t)->Projection->Projection",
"Dual->Projection->Projection",
"Dual->Projection->Projection",
"Join{DataScan(ta)->DataScan(tb)}(test.t.d,test.t.b)(test.t.a,test.t.c)->Projection",
"Join{DataScan(t1)->DataScan(t2)}(test.t.a,test.t.b)(test.t.d,test.t.d)->Projection",
"Join{DataScan(ta)->DataScan(tb)}(test.t.d,test.t.d)->Projection",
Expand Down
Loading

0 comments on commit 0ac6d80

Please sign in to comment.