planner: split casetest for pushdown (#45478)

ref pingcap/tidb#44940
This commit is contained in:
Weizhen Wang
2023-07-21 10:18:47 +08:00
committed by GitHub
parent c3f6004460
commit 20df6ffdf4
10 changed files with 1332 additions and 1238 deletions

View File

@ -1,6 +1,7 @@
startup --host_jvm_args=-Xmx4g
startup --unlimit_coredumps
dump: --http_connector_attempts=2
dump: --http_connector_retry_max_timeout=5s
run:ci --color=yes
build --announce_rc
@ -17,7 +18,6 @@ build:release --workspace_status_command=./build/print-workspace-status.sh --sta
build:release --config=ci
build:race --config=ci
build:race --@io_bazel_rules_go//go/config:race --test_env=GORACE=halt_on_error=1 --test_sharding_strategy=disabled
test --test_env=TZ=Asia/Shanghai
test --test_output=errors --test_summary=detailed
test:ci --color=yes

View File

@ -15,7 +15,6 @@ go_test(
],
data = glob(["testdata/**"]),
flaky = True,
race = "on",
shard_count = 50,
deps = [
"//domain",
@ -24,7 +23,6 @@ go_test(
"//planner/core",
"//planner/property",
"//session",
"//sessionctx/stmtctx",
"//sessionctx/variable",
"//statistics/handle",
"//testkit",

View File

@ -23,7 +23,6 @@ import (
"github.com/pingcap/tidb/domain"
"github.com/pingcap/tidb/parser/model"
"github.com/pingcap/tidb/session"
"github.com/pingcap/tidb/sessionctx/stmtctx"
"github.com/pingcap/tidb/sessionctx/variable"
"github.com/pingcap/tidb/statistics/handle"
"github.com/pingcap/tidb/testkit"
@ -32,36 +31,6 @@ import (
"github.com/stretchr/testify/require"
)
func TestPushLimitDownIndexLookUpReader(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("set @@session.tidb_executor_concurrency = 4;")
tk.MustExec("set @@session.tidb_hash_join_concurrency = 5;")
tk.MustExec("set @@session.tidb_distsql_scan_concurrency = 15;")
tk.MustExec("use test")
tk.MustExec("drop table if exists tbl")
tk.MustExec("create table tbl(a int, b int, c int, key idx_b_c(b,c))")
tk.MustExec("insert into tbl values(1,1,1),(2,2,2),(3,3,3),(4,4,4),(5,5,5)")
tk.MustExec("analyze table tbl")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
tk.MustQuery(tt).Check(testkit.Rows(output[i].Plan...))
}
}
func TestAggColumnPrune(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
@ -136,50 +105,6 @@ func TestSimplifyOuterJoinWithCast(t *testing.T) {
}
}
func TestSelPushDownTiFlash(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int primary key, b varchar(20))")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 0")
tk.MustExec("set tidb_cost_model_version=2")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestVerboseExplain(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
@ -237,151 +162,6 @@ func TestVerboseExplain(t *testing.T) {
}
}
func TestPushDownToTiFlashWithKeepOrder(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int primary key, b varchar(20))")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 0")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestPushDownToTiFlashWithKeepOrderInFastMode(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int primary key, b varchar(20))")
tk.MustExec("set @@session.tiflash_fastscan=ON")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 0")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestJoinNotSupportedByTiFlash(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists table_1")
tk.MustExec("create table table_1(id int not null, bit_col bit(2) not null, datetime_col datetime not null)")
tk.MustExec("insert into table_1 values(1,b'1','2020-01-01 00:00:00'),(2,b'0','2020-01-01 00:00:00')")
tk.MustExec("analyze table table_1")
tk.MustExec("insert into mysql.expr_pushdown_blacklist values('dayofmonth', 'tiflash', '');")
tk.MustExec("admin reload expr_pushdown_blacklist;")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "table_1" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 1")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
tk.MustExec("set @@session.tidb_broadcast_join_threshold_size = 1")
tk.MustExec("set @@session.tidb_broadcast_join_threshold_count = 1")
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestIsolationReadTiFlashNotChoosePointGet(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
@ -863,50 +643,6 @@ func TestIssue23887(t *testing.T) {
tk.MustQuery("select count(1) from (select count(1) from (select * from t1 where c3 = 100) k) k2;").Check(testkit.Rows("1"))
}
func TestPushDownProjectionForTiFlash(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t (id int, value decimal(6,3), name char(128))")
tk.MustExec("analyze table t")
tk.MustExec("set session tidb_allow_mpp=OFF")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestReorderSimplifiedOuterJoins(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
@ -1000,74 +736,6 @@ func TestMergeContinuousSelections(t *testing.T) {
}
}
func TestPushDownProjectionForTiKV(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t (a int, b real, i int, id int, value decimal(6,3), name char(128), d decimal(6,3), s char(128), t datetime, c bigint as ((a+1)) virtual, e real as ((b+a)))")
tk.MustExec("analyze table t")
tk.MustExec("set session tidb_opt_projection_push_down=1")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestPushDownProjectionForTiFlashCoprocessor(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t (a int, b real, i int, id int, value decimal(6,3), name char(128), d decimal(6,3), s char(128), t datetime, c bigint as ((a+1)) virtual, e real as ((b+a)))")
tk.MustExec("analyze table t")
tk.MustExec("set session tidb_opt_projection_push_down=1")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestLimitIndexLookUpKeepOrder(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
@ -1533,67 +1201,6 @@ func TestGroupBySetVar(t *testing.T) {
}
}
func TestPushDownGroupConcatToTiFlash(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists ts")
tk.MustExec("create table ts (col_0 char(64), col_1 varchar(64) not null, col_2 varchar(1), id int primary key);")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "ts" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set @@tidb_isolation_read_engines='tiflash,tidb'; set @@tidb_allow_mpp=1; set @@tidb_enforce_mpp=1;")
var input []string
var output []struct {
SQL string
Plan []string
Warning []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
comment := fmt.Sprintf("case:%v sql:%s", i, tt)
warnings := tk.Session().GetSessionVars().StmtCtx.GetWarnings()
testdata.OnRecord(func() {
if len(warnings) > 0 {
output[i].Warning = make([]string, len(warnings))
for j, warning := range warnings {
output[i].Warning[j] = warning.Err.Error()
}
}
})
if len(output[i].Warning) == 0 {
require.Len(t, warnings, 0, comment)
} else {
require.Len(t, warnings, len(output[i].Warning), comment)
for j, warning := range warnings {
require.Equal(t, stmtctx.WarnLevelWarning, warning.Level, comment)
require.EqualError(t, warning.Err, output[i].Warning[j], comment)
}
}
}
}
func TestIndexMergeWithCorrelatedColumns(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)

View File

@ -0,0 +1,24 @@
load("@io_bazel_rules_go//go:def.bzl", "go_test")
go_test(
name = "pushdown_test",
timeout = "short",
srcs = [
"main_test.go",
"push_down_test.go",
],
data = glob(["testdata/**"]),
flaky = True,
race = "on",
shard_count = 8,
deps = [
"//domain",
"//parser/model",
"//testkit",
"//testkit/testdata",
"//testkit/testmain",
"//testkit/testsetup",
"@com_github_stretchr_testify//require",
"@org_uber_go_goleak//:goleak",
],
)

View File

@ -0,0 +1,54 @@
// 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 pushdown
import (
"flag"
"testing"
"github.com/pingcap/tidb/testkit/testdata"
"github.com/pingcap/tidb/testkit/testmain"
"github.com/pingcap/tidb/testkit/testsetup"
"go.uber.org/goleak"
)
var testDataMap = make(testdata.BookKeeper)
func TestMain(m *testing.M) {
testsetup.SetupForCommonTest()
flag.Parse()
testDataMap.LoadTestSuiteData("testdata", "integration_suite")
opts := []goleak.Option{
goleak.IgnoreTopFunction("github.com/golang/glog.(*fileSink).flushDaemon"),
goleak.IgnoreTopFunction("github.com/lestrrat-go/httprc.runFetchWorker"),
goleak.IgnoreTopFunction("go.etcd.io/etcd/client/pkg/v3/logutil.(*MergeLogger).outputLoop"),
goleak.IgnoreTopFunction("gopkg.in/natefinch/lumberjack%2ev2.(*Logger).millRun"),
goleak.IgnoreTopFunction("github.com/tikv/client-go/v2/txnkv/transaction.keepAlive"),
goleak.IgnoreTopFunction("go.opencensus.io/stats/view.(*worker).start"),
}
callback := func(i int) int {
testDataMap.GenerateOutputIfNeeded()
return i
}
goleak.VerifyTestMain(testmain.WrapTestingM(m, callback), opts...)
}
func GetIntegrationSuiteData() testdata.TestData {
return testDataMap["integration_suite"]
}

View File

@ -0,0 +1,356 @@
// 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 pushdown
import (
"testing"
"github.com/pingcap/tidb/domain"
"github.com/pingcap/tidb/parser/model"
"github.com/pingcap/tidb/testkit"
"github.com/pingcap/tidb/testkit/testdata"
"github.com/stretchr/testify/require"
)
func TestPushLimitDownIndexLookUpReader(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("set @@session.tidb_executor_concurrency = 4;")
tk.MustExec("set @@session.tidb_hash_join_concurrency = 5;")
tk.MustExec("set @@session.tidb_distsql_scan_concurrency = 15;")
tk.MustExec("use test")
tk.MustExec("drop table if exists tbl")
tk.MustExec("create table tbl(a int, b int, c int, key idx_b_c(b,c))")
tk.MustExec("insert into tbl values(1,1,1),(2,2,2),(3,3,3),(4,4,4),(5,5,5)")
tk.MustExec("analyze table tbl")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
tk.MustQuery(tt).Check(testkit.Rows(output[i].Plan...))
}
}
func TestPushDownToTiFlashWithKeepOrder(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int primary key, b varchar(20))")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 0")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestPushDownToTiFlashWithKeepOrderInFastMode(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int primary key, b varchar(20))")
tk.MustExec("set @@session.tiflash_fastscan=ON")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 0")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestPushDownProjectionForTiFlash(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t (id int, value decimal(6,3), name char(128))")
tk.MustExec("analyze table t")
tk.MustExec("set session tidb_allow_mpp=OFF")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestPushDownProjectionForTiKV(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t (a int, b real, i int, id int, value decimal(6,3), name char(128), d decimal(6,3), s char(128), t datetime, c bigint as ((a+1)) virtual, e real as ((b+a)))")
tk.MustExec("analyze table t")
tk.MustExec("set session tidb_opt_projection_push_down=1")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestPushDownProjectionForTiFlashCoprocessor(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("set tidb_cost_model_version=2")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t (a int, b real, i int, id int, value decimal(6,3), name char(128), d decimal(6,3), s char(128), t datetime, c bigint as ((a+1)) virtual, e real as ((b+a)))")
tk.MustExec("analyze table t")
tk.MustExec("set session tidb_opt_projection_push_down=1")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestSelPushDownTiFlash(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int primary key, b varchar(20))")
// since allow-mpp is adjusted to false, there will be no physical plan if TiFlash cop is banned.
tk.MustExec("set @@session.tidb_allow_tiflash_cop=ON")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 0")
tk.MustExec("set tidb_cost_model_version=2")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}
func TestJoinNotSupportedByTiFlash(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("use test")
tk.MustExec("drop table if exists table_1")
tk.MustExec("create table table_1(id int not null, bit_col bit(2) not null, datetime_col datetime not null)")
tk.MustExec("insert into table_1 values(1,b'1','2020-01-01 00:00:00'),(2,b'0','2020-01-01 00:00:00')")
tk.MustExec("analyze table table_1")
tk.MustExec("insert into mysql.expr_pushdown_blacklist values('dayofmonth', 'tiflash', '');")
tk.MustExec("admin reload expr_pushdown_blacklist;")
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Session())
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
require.True(t, exists)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "table_1" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
tk.MustExec("set @@session.tidb_isolation_read_engines = 'tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp = 1")
var input []string
var output []struct {
SQL string
Plan []string
}
integrationSuiteData := GetIntegrationSuiteData()
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
tk.MustExec("set @@session.tidb_broadcast_join_threshold_size = 1")
tk.MustExec("set @@session.tidb_broadcast_join_threshold_count = 1")
integrationSuiteData.LoadTestCases(t, &input, &output)
for i, tt := range input {
testdata.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = testdata.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
}
}

View File

@ -0,0 +1,112 @@
[
{
"name": "TestPushLimitDownIndexLookUpReader",
"cases": [
// Limit should be pushed down into IndexLookUpReader, row count of IndexLookUpReader and TableScan should be 1.00.
"explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 limit 2,1",
// Projection atop IndexLookUpReader, Limit should be pushed down into IndexLookUpReader, and Projection should have row count 1.00 as well.
"explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 order by b desc limit 2,1",
// Limit should be pushed down into IndexLookUpReader when Selection on top of IndexScan.
"explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 and c > 1 limit 2,1",
// Limit should NOT be pushed down into IndexLookUpReader when Selection on top of TableScan.
"explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 and a > 1 limit 2,1"
]
},
{
"name": "TestPushDownToTiFlashWithKeepOrder",
"cases": [
"explain format = 'brief' select max(a) from t",
"explain format = 'brief' select min(a) from t"
]
},
{
"name": "TestPushDownToTiFlashWithKeepOrderInFastMode",
"cases": [
"explain format = 'brief' select max(a) from t",
"explain format = 'brief' select min(a) from t"
]
},
{
"name": "TestPushDownProjectionForTiFlashCoprocessor",
"cases": [
"desc format = 'brief' select i * 2 from t",
"desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"desc format = 'brief' select md5(s) from t; -- we do generate mpp plan, while the cost-cmp failed",
"desc format = 'brief' select c from t where a+1=3",
"desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;"
]
},
{
"name": "TestPushDownProjectionForTiFlash",
"cases": [
"desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;"
]
},
{
"name": "TestSelPushDownTiFlash",
"cases": [
"explain format = 'brief' select * from t where t.a > 1 and t.b = \"flash\" or t.a + 3 * t.a = 5",
"explain format = 'brief' select * from t where cast(t.a as double) + 3 = 5.1",
"explain format = 'brief' select * from t where b > 'a' order by convert(b, unsigned) limit 2",
"explain format = 'brief' select * from t where b > 'a' order by b limit 2"
]
},
{
"name": "TestPushDownProjectionForTiKV",
"cases": [
"desc format = 'brief' select i * 2 from t",
"desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"desc format = 'brief' select md5(s) from t",
"desc format = 'brief' select c from t where a+1=3",
"desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;"
]
},
{
"name": "TestJoinNotSupportedByTiFlash",
"cases": [
"explain format = 'brief' select * from table_1 a, table_1 b where a.bit_col = b.bit_col",
"explain format = 'brief' select * from table_1 a left join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > 100",
"explain format = 'brief' select * from table_1 a right join table_1 b on a.id = b.id and dayofmonth(b.datetime_col) > 100",
"explain format = 'brief' select * from table_1 a join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > dayofmonth(b.datetime_col)"
]
}
]

View File

@ -0,0 +1,758 @@
[
{
"Name": "TestPushLimitDownIndexLookUpReader",
"Cases": [
{
"SQL": "explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 limit 2,1",
"Plan": [
"IndexLookUp 1.00 root limit embedded(offset:2, count:1)",
"├─Limit(Build) 3.00 cop[tikv] offset:0, count:3",
"│ └─IndexRangeScan 3.00 cop[tikv] table:tbl, index:idx_b_c(b, c) range:(1,+inf], keep order:false",
"└─TableRowIDScan(Probe) 1.00 cop[tikv] table:tbl keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 order by b desc limit 2,1",
"Plan": [
"Projection 1.00 root test.tbl.a, test.tbl.b, test.tbl.c",
"└─IndexLookUp 1.00 root limit embedded(offset:2, count:1)",
" ├─Limit(Build) 3.00 cop[tikv] offset:0, count:3",
" │ └─IndexRangeScan 3.00 cop[tikv] table:tbl, index:idx_b_c(b, c) range:(1,+inf], keep order:true, desc",
" └─TableRowIDScan(Probe) 1.00 cop[tikv] table:tbl keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 and c > 1 limit 2,1",
"Plan": [
"IndexLookUp 1.00 root limit embedded(offset:2, count:1)",
"├─Limit(Build) 3.00 cop[tikv] offset:0, count:3",
"│ └─Selection 3.00 cop[tikv] gt(test.tbl.c, 1)",
"│ └─IndexRangeScan 3.75 cop[tikv] table:tbl, index:idx_b_c(b, c) range:(1,+inf], keep order:false",
"└─TableRowIDScan(Probe) 1.00 cop[tikv] table:tbl keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from tbl use index(idx_b_c) where b > 1 and a > 1 limit 2,1",
"Plan": [
"Limit 1.00 root offset:2, count:1",
"└─IndexLookUp 3.00 root ",
" ├─IndexRangeScan(Build) 3.75 cop[tikv] table:tbl, index:idx_b_c(b, c) range:(1,+inf], keep order:false",
" └─Limit(Probe) 3.00 cop[tikv] offset:0, count:3",
" └─Selection 3.00 cop[tikv] gt(test.tbl.a, 1)",
" └─TableRowIDScan 3.75 cop[tikv] table:tbl keep order:false"
]
}
]
},
{
"Name": "TestPushDownToTiFlashWithKeepOrder",
"Cases": [
{
"SQL": "explain format = 'brief' select max(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:max(test.t.a)->Column#3",
"└─TopN 1.00 root test.t.a:desc, offset:0, count:1",
" └─TableReader 1.00 root data:TopN",
" └─TopN 1.00 batchCop[tiflash] test.t.a:desc, offset:0, count:1",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select min(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:min(test.t.a)->Column#3",
"└─Limit 1.00 root offset:0, count:1",
" └─TableReader 1.00 root data:Limit",
" └─Limit 1.00 cop[tiflash] offset:0, count:1",
" └─TableFullScan 1.00 cop[tiflash] table:t keep order:true, stats:pseudo"
]
}
]
},
{
"Name": "TestPushDownToTiFlashWithKeepOrderInFastMode",
"Cases": [
{
"SQL": "explain format = 'brief' select max(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:max(test.t.a)->Column#3",
"└─TopN 1.00 root test.t.a:desc, offset:0, count:1",
" └─TableReader 1.00 root data:TopN",
" └─TopN 1.00 batchCop[tiflash] test.t.a:desc, offset:0, count:1",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select min(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:min(test.t.a)->Column#3",
"└─TopN 1.00 root test.t.a, offset:0, count:1",
" └─TableReader 1.00 root data:TopN",
" └─TopN 1.00 batchCop[tiflash] test.t.a, offset:0, count:1",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestPushDownProjectionForTiFlashCoprocessor",
"Cases": [
{
"SQL": "desc format = 'brief' select i * 2 from t",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] mul(test.t.i, 2)->Column#13",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] date_format(test.t.t, %Y-%m-%d %H)->Column#13",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select md5(s) from t; -- we do generate mpp plan, while the cost-cmp failed",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] md5(test.t.s)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select c from t where a+1=3",
"Plan": [
"Projection 8000.00 root test.t.c",
"└─TableReader 8000.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Selection 8000.00 mpp[tiflash] eq(plus(test.t.a, 1), 3)",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#17)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(Column#19)->Column#17",
" └─Projection 10000.00 mpp[tiflash] plus(test.t.id, 1)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#16)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(test.t._tidb_rowid)->Column#16",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#17)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:sum(Column#19)->Column#17",
" └─Projection 10000.00 mpp[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#18)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(Column#19)->Column#18",
" └─Projection 10000.00 mpp[tiflash] plus(test.t.id, 1)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#17)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(test.t._tidb_rowid)->Column#17",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#18)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:sum(Column#19)->Column#18",
" └─Projection 10000.00 mpp[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashJoin 10000.00 mpp[tiflash] inner join, equal:[eq(Column#13, Column#26)]",
" ├─ExchangeReceiver(Build) 8000.00 mpp[tiflash] ",
" │ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#13",
" │ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Projection(Probe) 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#26",
" └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 9990.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Selection 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root left outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 12487.50 root right outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 10000.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─Projection 10000.00 mpp[tiflash] minus(test.t.id, 2)->Column#25",
"│ └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 9990.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Selection 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] Column#26, Column#13",
" └─HashJoin 10000.00 mpp[tiflash] inner join, equal:[eq(Column#13, Column#26)]",
" ├─ExchangeReceiver(Build) 8000.00 mpp[tiflash] ",
" │ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#13",
" │ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Projection(Probe) 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#26",
" └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"Plan": [
"TableReader 7992.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 7992.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashJoin 7992.00 mpp[tiflash] semi join, equal:[eq(test.t.id, test.t.id)]",
" ├─ExchangeReceiver(Build) 9990.00 mpp[tiflash] ",
" │ └─ExchangeSender 9990.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─Selection 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Selection(Probe) 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 mpp[tiflash] table:A pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"Plan": [
"TableReader 8000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashJoin 8000.00 mpp[tiflash] anti semi join, equal:[eq(test.t.id, test.t.id)]",
" ├─ExchangeReceiver(Build) 10000.00 mpp[tiflash] ",
" │ └─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo",
" └─TableFullScan(Probe) 10000.00 mpp[tiflash] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] from_unixtime(cast(test.t.name, decimal(65,6) BINARY), %Y-%m-%d)->Column#13",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestPushDownProjectionForTiFlash",
"Cases": [
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#8)->Column#6",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 batchCop[tiflash] funcs:count(Column#9)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] plus(test.t.id, 1)->Column#9",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#7)->Column#6",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 batchCop[tiflash] funcs:count(test.t._tidb_rowid)->Column#7",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#8)->Column#6",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 batchCop[tiflash] funcs:sum(Column#9)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#9",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#8)->Column#6",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 batchCop[tiflash] funcs:count(Column#10)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] plus(test.t.id, 1)->Column#10",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#7)->Column#6",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 batchCop[tiflash] funcs:count(test.t._tidb_rowid)->Column#7",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:sum(Column#8)->Column#6",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 batchCop[tiflash] funcs:sum(Column#10)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#10",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(Column#5, Column#10)]",
"├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#10",
"│ └─TableReader 8000.00 root data:Selection",
"│ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─Projection(Probe) 8000.00 root minus(test.t.id, 2)->Column#5",
" └─TableReader 8000.00 root data:Selection",
" └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(test.t.id, Column#9)]",
"├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#9",
"│ └─TableReader 8000.00 root data:Selection",
"│ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root left outer join, equal:[eq(test.t.id, Column#9)]",
"├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#9",
"│ └─TableReader 8000.00 root data:Selection",
"│ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 12487.50 root right outer join, equal:[eq(test.t.id, Column#9)]",
"├─Projection(Build) 10000.00 root minus(test.t.id, 2)->Column#9",
"│ └─TableReader 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"Projection 10000.00 root Column#10, Column#5",
"└─HashJoin 10000.00 root inner join, equal:[eq(Column#5, Column#10)]",
" ├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#10",
" │ └─TableReader 8000.00 root data:Selection",
" │ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Projection(Probe) 8000.00 root minus(test.t.id, 2)->Column#5",
" └─TableReader 8000.00 root data:Selection",
" └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 7992.00 root semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tiflash] table:A pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 8000.00 root anti semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tiflash] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;",
"Plan": [
"Projection 10000.00 root from_unixtime(cast(test.t.name, decimal(65,6) BINARY), %Y-%m-%d)->Column#5",
"└─TableReader 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestSelPushDownTiFlash",
"Cases": [
{
"SQL": "explain format = 'brief' select * from t where t.a > 1 and t.b = \"flash\" or t.a + 3 * t.a = 5",
"Plan": [
"TableReader 8000.67 root data:Selection",
"└─Selection 8000.67 cop[tiflash] or(and(gt(test.t.a, 1), eq(test.t.b, \"flash\")), eq(plus(test.t.a, mul(3, test.t.a)), 5))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select * from t where cast(t.a as double) + 3 = 5.1",
"Plan": [
"TableReader 8000.00 root data:Selection",
"└─Selection 8000.00 cop[tiflash] eq(plus(cast(test.t.a, double BINARY), 3), 5.1)",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select * from t where b > 'a' order by convert(b, unsigned) limit 2",
"Plan": [
"Projection 2.00 root test.t.a, test.t.b",
"└─TopN 2.00 root Column#4, offset:0, count:2",
" └─Projection 2.00 root test.t.a, test.t.b, cast(test.t.b, bigint(22) UNSIGNED BINARY)->Column#4",
" └─TableReader 2.00 root data:Projection",
" └─Projection 2.00 batchCop[tiflash] test.t.a, test.t.b",
" └─TopN 2.00 batchCop[tiflash] Column#3, offset:0, count:2",
" └─Projection 3333.33 batchCop[tiflash] test.t.a, test.t.b, cast(test.t.b, bigint(22) UNSIGNED BINARY)->Column#3",
" └─Selection 3333.33 batchCop[tiflash] gt(test.t.b, \"a\")",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select * from t where b > 'a' order by b limit 2",
"Plan": [
"TopN 2.00 root test.t.b, offset:0, count:2",
"└─TableReader 2.00 root data:TopN",
" └─TopN 2.00 batchCop[tiflash] test.t.b, offset:0, count:2",
" └─Selection 3333.33 batchCop[tiflash] gt(test.t.b, \"a\")",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestPushDownProjectionForTiKV",
"Cases": [
{
"SQL": "desc format = 'brief' select i * 2 from t",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] mul(test.t.i, 2)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] date_format(test.t.t, %Y-%m-%d %H)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select md5(s) from t",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] md5(test.t.s)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select c from t where a+1=3",
"Plan": [
"Projection 8000.00 root test.t.c",
"└─TableReader 8000.00 root data:Selection",
" └─Selection 8000.00 cop[tikv] eq(plus(test.t.a, 1), 3)",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#16)->Column#14",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 cop[tikv] funcs:count(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#15)->Column#14",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 cop[tikv] funcs:count(1)->Column#15",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#16)->Column#14",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 cop[tikv] funcs:sum(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#16)->Column#14",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 cop[tikv] funcs:count(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#15)->Column#14",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 cop[tikv] funcs:count(1)->Column#15",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:sum(Column#16)->Column#14",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 cop[tikv] funcs:sum(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(Column#13, Column#26)]",
"├─TableReader(Build) 8000.00 root data:Projection",
"│ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#26",
"│ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 8000.00 root data:Projection",
" └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#13",
" └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root data:Projection",
"│ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root left outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root data:Projection",
"│ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 12487.50 root right outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 10000.00 root data:Projection",
"│ └─Projection 10000.00 cop[tikv] minus(test.t.id, 2)->Column#25",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"Projection 10000.00 root Column#26, Column#13",
"└─HashJoin 10000.00 root inner join, equal:[eq(Column#13, Column#26)]",
" ├─TableReader(Build) 8000.00 root data:Projection",
" │ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#26",
" │ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
" └─TableReader(Probe) 8000.00 root data:Projection",
" └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#13",
" └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 7992.00 root semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tikv] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 8000.00 root anti semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;",
"Plan": [
"Projection 10000.00 root from_unixtime(cast(test.t.name, decimal(65,6) BINARY), %Y-%m-%d)->Column#13",
"└─TableReader 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestJoinNotSupportedByTiFlash",
"Cases": [
{
"SQL": "explain format = 'brief' select * from table_1 a, table_1 b where a.bit_col = b.bit_col",
"Plan": [
"HashJoin 2.00 root inner join, equal:[eq(test.table_1.bit_col, test.table_1.bit_col)]",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from table_1 a left join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > 100",
"Plan": [
"HashJoin 2.00 root left outer join, equal:[eq(test.table_1.id, test.table_1.id)], left cond:[gt(dayofmonth(test.table_1.datetime_col), 100)]",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from table_1 a right join table_1 b on a.id = b.id and dayofmonth(b.datetime_col) > 100",
"Plan": [
"HashJoin 2.00 root right outer join, equal:[eq(test.table_1.id, test.table_1.id)], right cond:gt(dayofmonth(test.table_1.datetime_col), 100)",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from table_1 a join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > dayofmonth(b.datetime_col)",
"Plan": [
"HashJoin 2.00 root inner join, equal:[eq(test.table_1.id, test.table_1.id)], other cond:gt(dayofmonth(test.table_1.datetime_col), dayofmonth(test.table_1.datetime_col))",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false"
]
}
]
}
]

View File

@ -329,41 +329,36 @@
]
},
{
"name": "TestIssue30200",
"cases": [
// to_base64 and from_base64 has not been pushed to TiKV or TiFlash.
// We expect a Selection will be added above IndexMerge.
"select /*+ use_index_merge(t1) */ 1 from t1 where c1 = 'de' or c2 = '10' and from_base64(to_base64(c1)) = 'ab';",
// `left` has not been pushed to TiKV, but it has been pushed to TiFlash.
// We expect a Selection will be added above IndexMerge.
"select /*+ use_index_merge(t1) */ 1 from t1 where c1 = 'ab' or c2 = '10' and char_length(left(c1, 10)) = 10;",
// c3 is part of idx_1, so it will be put in partial_path's IndexFilters instead of TableFilters.
// But it still cannot be pushed to TiKV. This case cover code in DataSource.buildIndexMergeOrPath.
"select /*+ use_index_merge(tt1) */ 1 from tt1 where c1 = 'de' or c2 = '10' and from_base64(to_base64(c3)) = '10';",
// to_base64(left(pk, 5)) is in partial_path's TableFilters. But it cannot be pushed to TiKV.
// So it should be executed in TiDB. This case cover code in DataSource.buildIndexMergeOrPath.
"select /*+ use_index_merge( tt2 ) */ 1 from tt2 where tt2.c1 in (-3896405) or tt2.pk in (1, 53330) and to_base64(left(pk, 5));",
// This case covert expression index.
"select /*+ use_index_merge(tt3) */ 1 from tt3 where c1 < -10 or c2 < 10 and reverse(c3) = '2';",
// If no hint, we cannot use index merge if filter cannot be pushed to any storage.
"select 1 from t1 where c1 = 'de' or c2 = '10' and from_base64(to_base64(c1)) = 'ab';"
]
"name": "TestIssue30200",
"cases": [
// to_base64 and from_base64 has not been pushed to TiKV or TiFlash.
// We expect a Selection will be added above IndexMerge.
"select /*+ use_index_merge(t1) */ 1 from t1 where c1 = 'de' or c2 = '10' and from_base64(to_base64(c1)) = 'ab';",
// `left` has not been pushed to TiKV, but it has been pushed to TiFlash.
// We expect a Selection will be added above IndexMerge.
"select /*+ use_index_merge(t1) */ 1 from t1 where c1 = 'ab' or c2 = '10' and char_length(left(c1, 10)) = 10;",
// c3 is part of idx_1, so it will be put in partial_path's IndexFilters instead of TableFilters.
// But it still cannot be pushed to TiKV. This case cover code in DataSource.buildIndexMergeOrPath.
"select /*+ use_index_merge(tt1) */ 1 from tt1 where c1 = 'de' or c2 = '10' and from_base64(to_base64(c3)) = '10';",
// to_base64(left(pk, 5)) is in partial_path's TableFilters. But it cannot be pushed to TiKV.
// So it should be executed in TiDB. This case cover code in DataSource.buildIndexMergeOrPath.
"select /*+ use_index_merge( tt2 ) */ 1 from tt2 where tt2.c1 in (-3896405) or tt2.pk in (1, 53330) and to_base64(left(pk, 5));",
// This case covert expression index.
"select /*+ use_index_merge(tt3) */ 1 from tt3 where c1 < -10 or c2 < 10 and reverse(c3) = '2';",
// If no hint, we cannot use index merge if filter cannot be pushed to any storage.
"select 1 from t1 where c1 = 'de' or c2 = '10' and from_base64(to_base64(c1)) = 'ab';"
]
},
{
"name": "TestIndexMergeWithCorrelatedColumns",
"cases": [
"select * from t2 where c1 < all(select /*+ use_index_merge(t1) */ c1 from t1 where (c1 = 10 and c1 = t2.c3 or c2 = 1 and c2 = t2.c3) and substring(c3, 10)) order by c1;",
"select * from t2 where c1 < all(select /*+ use_index_merge(t1) */ c1 from t1 where (c1 = 10 and c1 = t2.c3 or c2 = 1 and c2 = t2.c3) and reverse(c3)) order by c1;",
"select * from t2 where c1 < all(select /*+ use_index_merge(t1) */ c1 from t1 where (c1 >= 10 and c1 = t2.c3 or c2 = 1 and c2 = t2.c3) and substring(c3, 10)) order by c1;",
// Test correlated column in IndexPath.TableFilters.
"select c_int from tt1 where c_decimal < all (select /*+ use_index_merge(tt2) */ c_decimal from tt2 where tt1.c_int = tt2.c_int and tt1.c_datetime > tt2.c_datetime and tt2.c_decimal = 9.060 or tt2.c_str <= 'interesting shtern' and tt1.c_int = tt2.c_int) order by 1;",
// Test correlated column in TablePath.TableFilters.
"select c_int from tt1 where c_decimal > all (select /*+ use_index_merge(tt2) */ c_decimal from tt2 where tt2.c_int = 7 and tt2.c_int < tt1.c_decimal or tt2.c_str >= 'zzzzzzzzzzzzzzzzzzz' and tt1.c_int = tt2.c_int) order by 1;"
"select * from t2 where c1 < all(select /*+ use_index_merge(t1) */ c1 from t1 where (c1 = 10 and c1 = t2.c3 or c2 = 1 and c2 = t2.c3) and substring(c3, 10)) order by c1;",
"select * from t2 where c1 < all(select /*+ use_index_merge(t1) */ c1 from t1 where (c1 = 10 and c1 = t2.c3 or c2 = 1 and c2 = t2.c3) and reverse(c3)) order by c1;",
"select * from t2 where c1 < all(select /*+ use_index_merge(t1) */ c1 from t1 where (c1 >= 10 and c1 = t2.c3 or c2 = 1 and c2 = t2.c3) and substring(c3, 10)) order by c1;",
// Test correlated column in IndexPath.TableFilters.
"select c_int from tt1 where c_decimal < all (select /*+ use_index_merge(tt2) */ c_decimal from tt2 where tt1.c_int = tt2.c_int and tt1.c_datetime > tt2.c_datetime and tt2.c_decimal = 9.060 or tt2.c_str <= 'interesting shtern' and tt1.c_int = tt2.c_int) order by 1;",
// Test correlated column in TablePath.TableFilters.
"select c_int from tt1 where c_decimal > all (select /*+ use_index_merge(tt2) */ c_decimal from tt2 where tt2.c_int = 7 and tt2.c_int < tt1.c_decimal or tt2.c_str >= 'zzzzzzzzzzzzzzzzzzz' and tt1.c_int = tt2.c_int) order by 1;"
]
},
{
@ -374,15 +369,6 @@
"explain format = 'brief' select count(*) from t31240;"
]
},
{
"name": "TestSelPushDownTiFlash",
"cases": [
"explain format = 'brief' select * from t where t.a > 1 and t.b = \"flash\" or t.a + 3 * t.a = 5",
"explain format = 'brief' select * from t where cast(t.a as double) + 3 = 5.1",
"explain format = 'brief' select * from t where b > 'a' order by convert(b, unsigned) limit 2",
"explain format = 'brief' select * from t where b > 'a' order by b limit 2"
]
},
{
"name": "TestVerboseExplain",
"cases": [
@ -402,7 +388,6 @@
"explain format = 'verbose' select (2) in (select /*+ read_from_storage(tiflash[t1]) */ count(*) from t1) from (select t.b < (select /*+ read_from_storage(tiflash[t2]) */ t.b from t2 limit 1 ) from t3 t) t; -- we do generate the agg pushed-down plan of mpp, but cost-cmp failed",
"explain format = 'verbose' select /*+ merge_join(t1), read_from_storage(tiflash[t1, t2]) */ count(*) from t1 join t2 on t1.a = t2.a"
]
},
{
"name": "TestRegardNULLAsPoint",
@ -423,29 +408,6 @@
"select * from tik where a<=>null and b<=>null and c<=>null"
]
},
{
"name": "TestPushDownToTiFlashWithKeepOrder",
"cases": [
"explain format = 'brief' select max(a) from t",
"explain format = 'brief' select min(a) from t"
]
},
{
"name": "TestPushDownToTiFlashWithKeepOrderInFastMode",
"cases": [
"explain format = 'brief' select max(a) from t",
"explain format = 'brief' select min(a) from t"
]
},
{
"name": "TestJoinNotSupportedByTiFlash",
"cases": [
"explain format = 'brief' select * from table_1 a, table_1 b where a.bit_col = b.bit_col",
"explain format = 'brief' select * from table_1 a left join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > 100",
"explain format = 'brief' select * from table_1 a right join table_1 b on a.id = b.id and dayofmonth(b.datetime_col) > 100",
"explain format = 'brief' select * from table_1 a join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > dayofmonth(b.datetime_col)"
]
},
{
"name": "TestIsolationReadDoNotFilterSystemDB",
"cases": [
@ -472,71 +434,6 @@
"explain format = 'brief' select /*+ inl_hash_join(s) */ * from t join s on t.a=s.a and t.a = s.b"
]
},
{
"name": "TestPushDownProjectionForTiKV",
"cases": [
"desc format = 'brief' select i * 2 from t",
"desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"desc format = 'brief' select md5(s) from t",
"desc format = 'brief' select c from t where a+1=3",
"desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;"
]
},
{
"name": "TestPushDownProjectionForTiFlashCoprocessor",
"cases": [
"desc format = 'brief' select i * 2 from t",
"desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"desc format = 'brief' select md5(s) from t; -- we do generate mpp plan, while the cost-cmp failed",
"desc format = 'brief' select c from t where a+1=3",
"desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;"
]
},
{
"name": "TestPushDownProjectionForTiFlash",
"cases": [
"desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;"
]
},
{
"name": "TestIndexMergeSerial",
"cases": [
@ -646,7 +543,6 @@
"explain format = 'brief' select row_number() over w1 from t1 a join t1 b on a.c1 = b.c2 window w1 as (partition by a.c1);",
// Selection.
"explain format = 'brief' select row_number() over w1 from t1 where c1 < 100 window w1 as (partition by c1 order by c1);",
// 2. Cannot use fine grained shuffle.
// No window function, so disabled.
"explain format = 'brief' select * from t1;",

View File

@ -2018,51 +2018,6 @@
}
]
},
{
"Name": "TestSelPushDownTiFlash",
"Cases": [
{
"SQL": "explain format = 'brief' select * from t where t.a > 1 and t.b = \"flash\" or t.a + 3 * t.a = 5",
"Plan": [
"TableReader 8000.67 root data:Selection",
"└─Selection 8000.67 cop[tiflash] or(and(gt(test.t.a, 1), eq(test.t.b, \"flash\")), eq(plus(test.t.a, mul(3, test.t.a)), 5))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select * from t where cast(t.a as double) + 3 = 5.1",
"Plan": [
"TableReader 8000.00 root data:Selection",
"└─Selection 8000.00 cop[tiflash] eq(plus(cast(test.t.a, double BINARY), 3), 5.1)",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select * from t where b > 'a' order by convert(b, unsigned) limit 2",
"Plan": [
"Projection 2.00 root test.t.a, test.t.b",
"└─TopN 2.00 root Column#4, offset:0, count:2",
" └─Projection 2.00 root test.t.a, test.t.b, cast(test.t.b, bigint(22) UNSIGNED BINARY)->Column#4",
" └─TableReader 2.00 root data:Projection",
" └─Projection 2.00 batchCop[tiflash] test.t.a, test.t.b",
" └─TopN 2.00 batchCop[tiflash] Column#3, offset:0, count:2",
" └─Projection 3333.33 batchCop[tiflash] test.t.a, test.t.b, cast(test.t.b, bigint(22) UNSIGNED BINARY)->Column#3",
" └─Selection 3333.33 batchCop[tiflash] gt(test.t.b, \"a\")",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select * from t where b > 'a' order by b limit 2",
"Plan": [
"TopN 2.00 root test.t.b, offset:0, count:2",
"└─TableReader 2.00 root data:TopN",
" └─TopN 2.00 batchCop[tiflash] test.t.b, offset:0, count:2",
" └─Selection 3333.33 batchCop[tiflash] gt(test.t.b, \"a\")",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestVerboseExplain",
"Cases": [
@ -2478,109 +2433,6 @@
}
]
},
{
"Name": "TestPushDownToTiFlashWithKeepOrder",
"Cases": [
{
"SQL": "explain format = 'brief' select max(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:max(test.t.a)->Column#3",
"└─TopN 1.00 root test.t.a:desc, offset:0, count:1",
" └─TableReader 1.00 root data:TopN",
" └─TopN 1.00 batchCop[tiflash] test.t.a:desc, offset:0, count:1",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select min(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:min(test.t.a)->Column#3",
"└─Limit 1.00 root offset:0, count:1",
" └─TableReader 1.00 root data:Limit",
" └─Limit 1.00 cop[tiflash] offset:0, count:1",
" └─TableFullScan 1.00 cop[tiflash] table:t keep order:true, stats:pseudo"
]
}
]
},
{
"Name": "TestPushDownToTiFlashWithKeepOrderInFastMode",
"Cases": [
{
"SQL": "explain format = 'brief' select max(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:max(test.t.a)->Column#3",
"└─TopN 1.00 root test.t.a:desc, offset:0, count:1",
" └─TableReader 1.00 root data:TopN",
" └─TopN 1.00 batchCop[tiflash] test.t.a:desc, offset:0, count:1",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "explain format = 'brief' select min(a) from t",
"Plan": [
"StreamAgg 1.00 root funcs:min(test.t.a)->Column#3",
"└─TopN 1.00 root test.t.a, offset:0, count:1",
" └─TableReader 1.00 root data:TopN",
" └─TopN 1.00 batchCop[tiflash] test.t.a, offset:0, count:1",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestJoinNotSupportedByTiFlash",
"Cases": [
{
"SQL": "explain format = 'brief' select * from table_1 a, table_1 b where a.bit_col = b.bit_col",
"Plan": [
"HashJoin 2.00 root inner join, equal:[eq(test.table_1.bit_col, test.table_1.bit_col)]",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from table_1 a left join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > 100",
"Plan": [
"HashJoin 2.00 root left outer join, equal:[eq(test.table_1.id, test.table_1.id)], left cond:[gt(dayofmonth(test.table_1.datetime_col), 100)]",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from table_1 a right join table_1 b on a.id = b.id and dayofmonth(b.datetime_col) > 100",
"Plan": [
"HashJoin 2.00 root right outer join, equal:[eq(test.table_1.id, test.table_1.id)], right cond:gt(dayofmonth(test.table_1.datetime_col), 100)",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false"
]
},
{
"SQL": "explain format = 'brief' select * from table_1 a join table_1 b on a.id = b.id and dayofmonth(a.datetime_col) > dayofmonth(b.datetime_col)",
"Plan": [
"HashJoin 2.00 root inner join, equal:[eq(test.table_1.id, test.table_1.id)], other cond:gt(dayofmonth(test.table_1.datetime_col), dayofmonth(test.table_1.datetime_col))",
"├─TableReader(Build) 2.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─TableFullScan 2.00 mpp[tiflash] table:b keep order:false",
"└─TableReader(Probe) 2.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 2.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 2.00 mpp[tiflash] table:a keep order:false"
]
}
]
},
{
"Name": "TestIsolationReadDoNotFilterSystemDB",
"Cases": [
@ -2715,569 +2567,6 @@
}
]
},
{
"Name": "TestPushDownProjectionForTiKV",
"Cases": [
{
"SQL": "desc format = 'brief' select i * 2 from t",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] mul(test.t.i, 2)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] date_format(test.t.t, %Y-%m-%d %H)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select md5(s) from t",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] md5(test.t.s)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select c from t where a+1=3",
"Plan": [
"Projection 8000.00 root test.t.c",
"└─TableReader 8000.00 root data:Selection",
" └─Selection 8000.00 cop[tikv] eq(plus(test.t.a, 1), 3)",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#16)->Column#14",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 cop[tikv] funcs:count(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#15)->Column#14",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 cop[tikv] funcs:count(1)->Column#15",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#16)->Column#14",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 cop[tikv] funcs:sum(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#16)->Column#14",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 cop[tikv] funcs:count(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#15)->Column#14",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 cop[tikv] funcs:count(1)->Column#15",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:sum(Column#16)->Column#14",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 cop[tikv] funcs:sum(plus(test.t.id, 1))->Column#16",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(Column#13, Column#26)]",
"├─TableReader(Build) 8000.00 root data:Projection",
"│ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#26",
"│ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 8000.00 root data:Projection",
" └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#13",
" └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root data:Projection",
"│ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root left outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root data:Projection",
"│ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 12487.50 root right outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 10000.00 root data:Projection",
"│ └─Projection 10000.00 cop[tikv] minus(test.t.id, 2)->Column#25",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"Projection 10000.00 root Column#26, Column#13",
"└─HashJoin 10000.00 root inner join, equal:[eq(Column#13, Column#26)]",
" ├─TableReader(Build) 8000.00 root data:Projection",
" │ └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#26",
" │ └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
" └─TableReader(Probe) 8000.00 root data:Projection",
" └─Projection 8000.00 cop[tikv] minus(test.t.id, 2)->Column#13",
" └─Selection 8000.00 cop[tikv] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 7992.00 root semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tikv] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tikv] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 8000.00 root anti semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;",
"Plan": [
"Projection 10000.00 root from_unixtime(cast(test.t.name, decimal(65,6) BINARY), %Y-%m-%d)->Column#13",
"└─TableReader 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestPushDownProjectionForTiFlashCoprocessor",
"Cases": [
{
"SQL": "desc format = 'brief' select i * 2 from t",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] mul(test.t.i, 2)->Column#13",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select DATE_FORMAT(t, '%Y-%m-%d %H') as date from t",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] date_format(test.t.t, %Y-%m-%d %H)->Column#13",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select md5(s) from t; -- we do generate mpp plan, while the cost-cmp failed",
"Plan": [
"TableReader 10000.00 root data:Projection",
"└─Projection 10000.00 cop[tikv] md5(test.t.s)->Column#13",
" └─TableFullScan 10000.00 cop[tikv] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select c from t where a+1=3",
"Plan": [
"Projection 8000.00 root test.t.c",
"└─TableReader 8000.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Selection 8000.00 mpp[tiflash] eq(plus(test.t.a, 1), 3)",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#17)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(Column#19)->Column#17",
" └─Projection 10000.00 mpp[tiflash] plus(test.t.id, 1)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#16)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(test.t._tidb_rowid)->Column#16",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#17)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:sum(Column#19)->Column#17",
" └─Projection 10000.00 mpp[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#18)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(Column#19)->Column#18",
" └─Projection 10000.00 mpp[tiflash] plus(test.t.id, 1)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#17)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:count(test.t._tidb_rowid)->Column#17",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#18)->Column#14",
"└─TableReader 1.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 1.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashAgg 1.00 mpp[tiflash] funcs:sum(Column#19)->Column#18",
" └─Projection 10000.00 mpp[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#19",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashJoin 10000.00 mpp[tiflash] inner join, equal:[eq(Column#13, Column#26)]",
" ├─ExchangeReceiver(Build) 8000.00 mpp[tiflash] ",
" │ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#13",
" │ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Projection(Probe) 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#26",
" └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 9990.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Selection 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root left outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 8000.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#25",
"│ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 12487.50 root right outer join, equal:[eq(test.t.id, Column#25)]",
"├─TableReader(Build) 10000.00 root MppVersion: 2, data:ExchangeSender",
"│ └─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
"│ └─Projection 10000.00 mpp[tiflash] minus(test.t.id, 2)->Column#25",
"│ └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root MppVersion: 2, data:ExchangeSender",
" └─ExchangeSender 9990.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Selection 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] Column#26, Column#13",
" └─HashJoin 10000.00 mpp[tiflash] inner join, equal:[eq(Column#13, Column#26)]",
" ├─ExchangeReceiver(Build) 8000.00 mpp[tiflash] ",
" │ └─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─Projection 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#13",
" │ └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Projection(Probe) 8000.00 mpp[tiflash] minus(test.t.id, 2)->Column#26",
" └─Selection 8000.00 mpp[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"Plan": [
"TableReader 7992.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 7992.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashJoin 7992.00 mpp[tiflash] semi join, equal:[eq(test.t.id, test.t.id)]",
" ├─ExchangeReceiver(Build) 9990.00 mpp[tiflash] ",
" │ └─ExchangeSender 9990.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─Selection 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Selection(Probe) 9990.00 mpp[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 mpp[tiflash] table:A pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"Plan": [
"TableReader 8000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 8000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─HashJoin 8000.00 mpp[tiflash] anti semi join, equal:[eq(test.t.id, test.t.id)]",
" ├─ExchangeReceiver(Build) 10000.00 mpp[tiflash] ",
" │ └─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: Broadcast, Compression: FAST",
" │ └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo",
" └─TableFullScan(Probe) 10000.00 mpp[tiflash] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;",
"Plan": [
"TableReader 10000.00 root MppVersion: 2, data:ExchangeSender",
"└─ExchangeSender 10000.00 mpp[tiflash] ExchangeType: PassThrough",
" └─Projection 10000.00 mpp[tiflash] from_unixtime(cast(test.t.name, decimal(65,6) BINARY), %Y-%m-%d)->Column#13",
" └─TableFullScan 10000.00 mpp[tiflash] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestPushDownProjectionForTiFlash",
"Cases": [
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#8)->Column#6",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 batchCop[tiflash] funcs:count(Column#9)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] plus(test.t.id, 1)->Column#9",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:count(Column#7)->Column#6",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 batchCop[tiflash] funcs:count(test.t._tidb_rowid)->Column#7",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ hash_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"HashAgg 1.00 root funcs:sum(Column#8)->Column#6",
"└─TableReader 1.00 root data:HashAgg",
" └─HashAgg 1.00 batchCop[tiflash] funcs:sum(Column#9)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#9",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#8)->Column#6",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 batchCop[tiflash] funcs:count(Column#10)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] plus(test.t.id, 1)->Column#10",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ count(*) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:count(Column#7)->Column#6",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 batchCop[tiflash] funcs:count(test.t._tidb_rowid)->Column#7",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select /*+ stream_agg()*/ sum(b) from (select /*+ read_from_storage(tiflash[t]) */ id + 1 as b from t)A",
"Plan": [
"StreamAgg 1.00 root funcs:sum(Column#8)->Column#6",
"└─TableReader 1.00 root data:StreamAgg",
" └─StreamAgg 1.00 batchCop[tiflash] funcs:sum(Column#10)->Column#8",
" └─Projection 10000.00 batchCop[tiflash] cast(plus(test.t.id, 1), decimal(20,0) BINARY)->Column#10",
" └─TableFullScan 10000.00 batchCop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(Column#5, Column#10)]",
"├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#10",
"│ └─TableReader 8000.00 root data:Selection",
"│ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─Projection(Probe) 8000.00 root minus(test.t.id, 2)->Column#5",
" └─TableReader 8000.00 root data:Selection",
" └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root inner join, equal:[eq(test.t.id, Column#9)]",
"├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#9",
"│ └─TableReader 8000.00 root data:Selection",
"│ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t left join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 10000.00 root left outer join, equal:[eq(test.t.id, Column#9)]",
"├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#9",
"│ └─TableReader 8000.00 root data:Selection",
"│ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select * from t right join (select id-2 as b from t) A on A.b=t.id",
"Plan": [
"HashJoin 12487.50 root right outer join, equal:[eq(test.t.id, Column#9)]",
"├─Projection(Build) 10000.00 root minus(test.t.id, 2)->Column#9",
"│ └─TableReader 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.b, B.b from (select id-2 as b from t) B join (select id-2 as b from t) A on A.b=B.b",
"Plan": [
"Projection 10000.00 root Column#10, Column#5",
"└─HashJoin 10000.00 root inner join, equal:[eq(Column#5, Column#10)]",
" ├─Projection(Build) 8000.00 root minus(test.t.id, 2)->Column#10",
" │ └─TableReader 8000.00 root data:Selection",
" │ └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
" │ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
" └─Projection(Probe) 8000.00 root minus(test.t.id, 2)->Column#5",
" └─TableReader 8000.00 root data:Selection",
" └─Selection 8000.00 cop[tiflash] not(isnull(minus(test.t.id, 2)))",
" └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 7992.00 root semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 9990.00 root data:Selection",
"│ └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t pushed down filter:empty, keep order:false, stats:pseudo",
"└─TableReader(Probe) 9990.00 root data:Selection",
" └─Selection 9990.00 cop[tiflash] not(isnull(test.t.id))",
" └─TableFullScan 10000.00 cop[tiflash] table:A pushed down filter:empty, keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' select A.id from t as A where not exists (select 1 from t where t.id=A.id)",
"Plan": [
"HashJoin 8000.00 root anti semi join, equal:[eq(test.t.id, test.t.id)]",
"├─TableReader(Build) 10000.00 root data:TableFullScan",
"│ └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo",
"└─TableReader(Probe) 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tiflash] table:A keep order:false, stats:pseudo"
]
},
{
"SQL": "desc format = 'brief' SELECT FROM_UNIXTIME(name,'%Y-%m-%d') FROM t;",
"Plan": [
"Projection 10000.00 root from_unixtime(cast(test.t.name, decimal(65,6) BINARY), %Y-%m-%d)->Column#5",
"└─TableReader 10000.00 root data:TableFullScan",
" └─TableFullScan 10000.00 cop[tiflash] table:t keep order:false, stats:pseudo"
]
}
]
},
{
"Name": "TestIndexMergeSerial",
"Cases": [