Skip to content

Commit

Permalink
planner: Revert tidb_allow_mpp modification for downgrade compatibi…
Browse files Browse the repository at this point in the history
…lity and add warnings for enforce mpp. (#25302) (#25353)
  • Loading branch information
ti-srebot committed Jun 11, 2021
1 parent 29bfbb2 commit 0dd2ca4
Show file tree
Hide file tree
Showing 16 changed files with 944 additions and 194 deletions.
2 changes: 2 additions & 0 deletions config/config.go
Original file line number Diff line number Diff line change
Expand Up @@ -424,6 +424,7 @@ type Performance struct {
MemProfileInterval string `toml:"mem-profile-interval" json:"mem-profile-interval"`
IndexUsageSyncLease string `toml:"index-usage-sync-lease" json:"index-usage-sync-lease"`
GOGC int `toml:"gogc" json:"gogc"`
EnforceMPP bool `toml:"enforce-mpp" json:"enforce-mpp"`
}

// PlanCache is the PlanCache section of the config.
Expand Down Expand Up @@ -623,6 +624,7 @@ var defaultConf = Config{
// TODO: set indexUsageSyncLease to 60s.
IndexUsageSyncLease: "0s",
GOGC: 100,
EnforceMPP: false,
},
ProxyProtocol: ProxyProtocol{
Networks: "",
Expand Down
8 changes: 4 additions & 4 deletions executor/tiflash_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -300,7 +300,7 @@ func (s *tiflashTestSuite) TestTiFlashPartitionTableShuffledHashJoin(c *C) {
tk.MustExec(fmt.Sprintf("analyze table %v", tbl))
}

tk.MustExec("SET tidb_allow_mpp=2")
tk.MustExec("SET tidb_enforce_mpp=1")
tk.MustExec("SET tidb_opt_broadcast_join=0")
tk.MustExec("SET tidb_broadcast_join_threshold_count=0")
tk.MustExec("SET tidb_broadcast_join_threshold_size=0")
Expand Down Expand Up @@ -378,7 +378,7 @@ func (s *tiflashTestSuite) TestTiFlashPartitionTableReader(c *C) {
tk.MustExec(fmt.Sprintf("insert into %v values %v", tbl, strings.Join(vals, ", ")))
}

tk.MustExec("SET tidb_allow_mpp=2")
tk.MustExec("SET tidb_enforce_mpp=1")
tk.MustExec("set @@session.tidb_isolation_read_engines='tiflash'")
for i := 0; i < 100; i++ {
l, r := rand.Intn(400), rand.Intn(400)
Expand Down Expand Up @@ -742,7 +742,7 @@ func (s *tiflashTestSuite) TestTiFlashPartitionTableShuffledHashAggregation(c *C
tk.MustExec(fmt.Sprintf("analyze table %v", tbl))
}
tk.MustExec("set @@session.tidb_isolation_read_engines='tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp=2")
tk.MustExec("set @@session.tidb_enforce_mpp=1")
// mock executor does not support use outer table as build side for outer join, so need to
// force the inner table as build side
tk.MustExec("set tidb_opt_mpp_outer_join_fixed_build_side=1")
Expand Down Expand Up @@ -814,7 +814,7 @@ func (s *tiflashTestSuite) TestTiFlashPartitionTableBroadcastJoin(c *C) {
tk.MustExec(fmt.Sprintf("analyze table %v", tbl))
}
tk.MustExec("set @@session.tidb_isolation_read_engines='tiflash'")
tk.MustExec("set @@session.tidb_allow_mpp=2")
tk.MustExec("set @@session.tidb_enforce_mpp=1")
tk.MustExec("set @@session.tidb_opt_broadcast_join=ON")
// mock executor does not support use outer table as build side for outer join, so need to
// force the inner table as build side
Expand Down
299 changes: 299 additions & 0 deletions planner/core/enforce_mpp_test.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,299 @@
// Copyright 2019 PingCAP, Inc.
//
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// See the License for the specific language governing permissions and
// limitations under the License.

package core_test

import (
"strings"

. "github.com/pingcap/check"
"github.com/pingcap/parser/model"
"github.com/pingcap/tidb/domain"
"github.com/pingcap/tidb/kv"
"github.com/pingcap/tidb/util/collate"
"github.com/pingcap/tidb/util/testkit"
"github.com/pingcap/tidb/util/testutil"
)

var _ = SerialSuites(&testEnforceMPPSuite{})

type testEnforceMPPSuite struct {
testData testutil.TestData
store kv.Storage
dom *domain.Domain
}

func (s *testEnforceMPPSuite) SetUpSuite(c *C) {
var err error
s.testData, err = testutil.LoadTestSuiteData("testdata", "enforce_mpp_suite")
c.Assert(err, IsNil)
}

func (s *testEnforceMPPSuite) TearDownSuite(c *C) {
c.Assert(s.testData.GenerateOutputIfNeeded(), IsNil)
}

func (s *testEnforceMPPSuite) SetUpTest(c *C) {
var err error
s.store, s.dom, err = newStoreWithBootstrap()
c.Assert(err, IsNil)
}

func (s *testEnforceMPPSuite) TearDownTest(c *C) {
s.dom.Close()
err := s.store.Close()
c.Assert(err, IsNil)
}

func (s *testEnforceMPPSuite) TestSetVariables(c *C) {
tk := testkit.NewTestKit(c, s.store)

// test value limit of tidb_opt_tiflash_concurrency_factor
err := tk.ExecToErr("set @@tidb_opt_tiflash_concurrency_factor = 0")
c.Assert(err, NotNil)
c.Assert(err.Error(), Equals, `[variable:1231]Variable 'tidb_opt_tiflash_concurrency_factor' can't be set to the value of '0'`)

// test set tidb_enforce_mpp when tidb_allow_mpp=false;
err = tk.ExecToErr("set @@tidb_allow_mpp = 0; set @@tidb_enforce_mpp = 1;")
c.Assert(err, NotNil)
c.Assert(err.Error(), Equals, `[variable:1231]Variable 'tidb_enforce_mpp' can't be set to the value of '1' but tidb_allow_mpp is 0, please activate tidb_allow_mpp at first.'`)

err = tk.ExecToErr("set @@tidb_allow_mpp = 1; set @@tidb_enforce_mpp = 1;")
c.Assert(err, IsNil)

err = tk.ExecToErr("set @@tidb_allow_mpp = 0;")
c.Assert(err, IsNil)
}

func (s *testEnforceMPPSuite) TestEnforceMPP(c *C) {
tk := testkit.NewTestKit(c, s.store)

// test query
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int, b int)")
tk.MustExec("create index idx on t(a)")

// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Se)
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
c.Assert(exists, IsTrue)
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
Warn []string
}
s.testData.GetTestCases(c, &input, &output)
for i, tt := range input {
s.testData.OnRecord(func() {
output[i].SQL = tt
})
if strings.HasPrefix(tt, "set") {
tk.MustExec(tt)
continue
}
s.testData.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = s.testData.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
output[i].Warn = s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
c.Assert(s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings()), DeepEquals, output[i].Warn)
}
}

// general cases.
func (s *testEnforceMPPSuite) TestEnforceMPPWarning1(c *C) {
tk := testkit.NewTestKit(c, s.store)

// test query
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("create table t(a int, b int as (a+1), c time)")
tk.MustExec("create index idx on t(a)")

var input []string
var output []struct {
SQL string
Plan []string
Warn []string
}
s.testData.GetTestCases(c, &input, &output)
for i, tt := range input {
s.testData.OnRecord(func() {
output[i].SQL = tt
})
if strings.HasPrefix(tt, "set") {
tk.MustExec(tt)
continue
}
if strings.HasPrefix(tt, "cmd: create-replica") {
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Se)
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
c.Assert(exists, IsTrue)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: false,
}
}
}
continue
}
if strings.HasPrefix(tt, "cmd: enable-replica") {
// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Se)
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
c.Assert(exists, IsTrue)
for _, tblInfo := range db.Tables {
if tblInfo.Name.L == "t" {
tblInfo.TiFlashReplica = &model.TiFlashReplicaInfo{
Count: 1,
Available: true,
}
}
}
continue
}
s.testData.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = s.testData.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
output[i].Warn = s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
c.Assert(s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings()), DeepEquals, output[i].Warn)
}
}

// partition table.
func (s *testEnforceMPPSuite) TestEnforceMPPWarning2(c *C) {
tk := testkit.NewTestKit(c, s.store)

// test query
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("CREATE TABLE t (a int, b char(20)) PARTITION BY HASH(a)")

// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Se)
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
c.Assert(exists, IsTrue)
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
Warn []string
}
s.testData.GetTestCases(c, &input, &output)
for i, tt := range input {
s.testData.OnRecord(func() {
output[i].SQL = tt
})
if strings.HasPrefix(tt, "set") {
tk.MustExec(tt)
continue
}
s.testData.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = s.testData.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
output[i].Warn = s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
c.Assert(s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings()), DeepEquals, output[i].Warn)
}
}

// new collation.
func (s *testEnforceMPPSuite) TestEnforceMPPWarning3(c *C) {
tk := testkit.NewTestKit(c, s.store)

// test query
tk.MustExec("use test")
tk.MustExec("drop table if exists t")
tk.MustExec("CREATE TABLE t (a int, b char(20))")

// Create virtual tiflash replica info.
dom := domain.GetDomain(tk.Se)
is := dom.InfoSchema()
db, exists := is.SchemaByName(model.NewCIStr("test"))
c.Assert(exists, IsTrue)
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
Warn []string
}
s.testData.GetTestCases(c, &input, &output)
for i, tt := range input {
s.testData.OnRecord(func() {
output[i].SQL = tt
})
if strings.HasPrefix(tt, "set") || strings.HasPrefix(tt, "UPDATE") {
tk.MustExec(tt)
continue
}
if strings.HasPrefix(tt, "cmd: enable-new-collation") {
collate.SetNewCollationEnabledForTest(true)
continue
}
if strings.HasPrefix(tt, "cmd: disable-new-collation") {
collate.SetNewCollationEnabledForTest(false)
continue
}
s.testData.OnRecord(func() {
output[i].SQL = tt
output[i].Plan = s.testData.ConvertRowsToStrings(tk.MustQuery(tt).Rows())
output[i].Warn = s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings())
})
res := tk.MustQuery(tt)
res.Check(testkit.Rows(output[i].Plan...))
c.Assert(s.testData.ConvertSQLWarnToStrings(tk.Se.GetSessionVars().StmtCtx.GetWarnings()), DeepEquals, output[i].Warn)
}
}
Loading

0 comments on commit 0dd2ca4

Please sign in to comment.