Skip to content

Commit

Permalink
Merge branch 'master' into ddl/fix-race-set-source
Browse files Browse the repository at this point in the history
  • Loading branch information
you06 committed Dec 22, 2022
2 parents 34ed712 + 0f4bd73 commit 3d118db
Show file tree
Hide file tree
Showing 52 changed files with 2,237 additions and 690 deletions.
4 changes: 2 additions & 2 deletions DEPS.bzl
Original file line number Diff line number Diff line change
Expand Up @@ -4467,8 +4467,8 @@ def go_deps():
name = "org_golang_x_time",
build_file_proto_mode = "disable_global",
importpath = "golang.org/x/time",
sum = "h1:52I/1L54xyEQAYdtcSuxtiT84KGYTBGXwayxmIpNJhE=",
version = "v0.2.0",
sum = "h1:rg5rLMjNzMS1RkNLzCG38eapWhnYLFYXDXj2gOlr8j4=",
version = "v0.3.0",
)
go_repository(
name = "org_golang_x_tools",
Expand Down
9 changes: 9 additions & 0 deletions bindinfo/handle.go
Original file line number Diff line number Diff line change
Expand Up @@ -533,6 +533,15 @@ func (h *BindHandle) SetBindRecordStatus(originalSQL string, binding *Binding, n
return
}

// SetBindRecordStatusByDigest set a BindRecord's status to the storage and bind cache.
func (h *BindHandle) SetBindRecordStatusByDigest(newStatus, sqlDigest string) (ok bool, err error) {
oldRecord, err := h.GetBindRecordBySQLDigest(sqlDigest)
if err != nil {
return false, err
}
return h.SetBindRecordStatus(oldRecord.OriginalSQL, nil, newStatus)
}

// GCBindRecord physically removes the deleted bind records in mysql.bind_info.
func (h *BindHandle) GCBindRecord() (err error) {
h.bindInfo.Lock()
Expand Down
7 changes: 1 addition & 6 deletions ddl/backfilling.go
Original file line number Diff line number Diff line change
Expand Up @@ -807,12 +807,7 @@ func (b *backfillScheduler) initCopReqSenderPool() {
logutil.BgLogger().Warn("[ddl-ingest] cannot init cop request sender", zap.Error(err))
return
}
ver, err := sessCtx.GetStore().CurrentVersion(kv.GlobalTxnScope)
if err != nil {
logutil.BgLogger().Warn("[ddl-ingest] cannot init cop request sender", zap.Error(err))
return
}
b.copReqSenderPool = newCopReqSenderPool(b.ctx, copCtx, ver.Ver)
b.copReqSenderPool = newCopReqSenderPool(b.ctx, copCtx, sessCtx.GetStore())
}

func (b *backfillScheduler) canSkipError(err error) bool {
Expand Down
12 changes: 10 additions & 2 deletions ddl/ddl_api.go
Original file line number Diff line number Diff line change
Expand Up @@ -1980,7 +1980,7 @@ func addIndexForForeignKey(ctx sessionctx.Context, tbInfo *model.TableInfo) erro
if handleCol != nil && len(fk.Cols) == 1 && handleCol.Name.L == fk.Cols[0].L {
continue
}
if model.FindIndexByColumns(tbInfo, fk.Cols...) != nil {
if model.FindIndexByColumns(tbInfo, tbInfo.Indices, fk.Cols...) != nil {
continue
}
idxName := fk.Name
Expand Down Expand Up @@ -3264,6 +3264,14 @@ func (d *ddl) AlterTable(ctx context.Context, sctx sessionctx.Context, stmt *ast
return dbterror.ErrOptOnCacheTable.GenWithStackByArgs("Alter Table")
}
}
// set name for anonymous foreign key.
maxForeignKeyID := tb.Meta().MaxForeignKeyID
for _, spec := range validSpecs {
if spec.Tp == ast.AlterTableAddConstraint && spec.Constraint.Tp == ast.ConstraintForeignKey && spec.Constraint.Name == "" {
maxForeignKeyID++
spec.Constraint.Name = fmt.Sprintf("fk_%d", maxForeignKeyID)
}
}

if len(validSpecs) > 1 {
sctx.GetSessionVars().StmtCtx.MultiSchemaInfo = model.NewMultiSchemaInfo()
Expand Down Expand Up @@ -6570,7 +6578,7 @@ func (d *ddl) CreateForeignKey(ctx sessionctx.Context, ti ast.Ident, fkName mode
if err != nil {
return err
}
if model.FindIndexByColumns(t.Meta(), fkInfo.Cols...) == nil {
if model.FindIndexByColumns(t.Meta(), t.Meta().Indices, fkInfo.Cols...) == nil {
// Need to auto create index for fk cols
if ctx.GetSessionVars().StmtCtx.MultiSchemaInfo == nil {
ctx.GetSessionVars().StmtCtx.MultiSchemaInfo = model.NewMultiSchemaInfo()
Expand Down
4 changes: 2 additions & 2 deletions ddl/export_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -28,15 +28,15 @@ func SetBatchInsertDeleteRangeSize(i int) {

var NewCopContext4Test = newCopContext

func FetchRowsFromCop4Test(copCtx *copContext, startKey, endKey kv.Key, startTS uint64,
func FetchRowsFromCop4Test(copCtx *copContext, startKey, endKey kv.Key, store kv.Storage,
batchSize int) ([]*indexRecord, bool, error) {
variable.SetDDLReorgBatchSize(int32(batchSize))
task := &reorgBackfillTask{
id: 1,
startKey: startKey,
endKey: endKey,
}
pool := newCopReqSenderPool(context.Background(), copCtx, startTS)
pool := newCopReqSenderPool(context.Background(), copCtx, store)
pool.adjustSize(1)
pool.tasksCh <- task
idxRec, _, _, done, err := pool.fetchRowColValsFromCop(*task)
Expand Down
29 changes: 29 additions & 0 deletions ddl/fktest/foreign_key_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -1562,3 +1562,32 @@ func getLatestSchemaDiff(t *testing.T, tk *testkit.TestKit) *model.SchemaDiff {
require.NoError(t, err)
return diff
}

func TestTestMultiSchemaAddForeignKey(t *testing.T) {
store, _ := testkit.CreateMockStoreAndDomain(t)
tk := testkit.NewTestKit(t, store)
tk.MustExec("set @@foreign_key_checks=1;")
tk.MustExec("use test")
tk.MustExec("create table t1 (id int key);")
tk.MustExec("create table t2 (a int, b int);")
tk.MustExec("alter table t2 add foreign key (a) references t1(id), add foreign key (b) references t1(id)")
tk.MustExec("alter table t2 add column c int, add column d int")
tk.MustExec("alter table t2 add foreign key (c) references t1(id), add foreign key (d) references t1(id), add index(c), add index(d)")
tk.MustExec("drop table t2")
tk.MustExec("create table t2 (a int, b int, index idx1(a), index idx2(b));")
tk.MustGetErrMsg("alter table t2 drop index idx1, drop index idx2, add foreign key (a) references t1(id), add foreign key (b) references t1(id)",
"[ddl:1553]Cannot drop index 'idx1': needed in a foreign key constraint")
tk.MustExec("alter table t2 drop index idx1, drop index idx2")
tk.MustExec("alter table t2 add foreign key (a) references t1(id), add foreign key (b) references t1(id)")
tk.MustQuery("show create table t2").Check(testkit.Rows("t2 CREATE TABLE `t2` (\n" +
" `a` int(11) DEFAULT NULL,\n" +
" `b` int(11) DEFAULT NULL,\n" +
" KEY `fk_1` (`a`),\n" +
" KEY `fk_2` (`b`),\n" +
" CONSTRAINT `fk_1` FOREIGN KEY (`a`) REFERENCES `test`.`t1` (`id`),\n" +
" CONSTRAINT `fk_2` FOREIGN KEY (`b`) REFERENCES `test`.`t1` (`id`)\n" +
") ENGINE=InnoDB DEFAULT CHARSET=utf8mb4 COLLATE=utf8mb4_bin"))
tk.MustExec("drop table t2")
tk.MustExec("create table t2 (a int, b int, index idx0(a,b), index idx1(a), index idx2(b));")
tk.MustExec("alter table t2 drop index idx1, add foreign key (a) references t1(id), add foreign key (b) references t1(id)")
}
4 changes: 2 additions & 2 deletions ddl/foreign_key.go
Original file line number Diff line number Diff line change
Expand Up @@ -291,7 +291,7 @@ func checkTableForeignKey(referTblInfo, tblInfo *model.TableInfo, fkInfo *model.
}
}
// check refer columns should have index.
if model.FindIndexByColumns(referTblInfo, fkInfo.RefCols...) == nil {
if model.FindIndexByColumns(referTblInfo, referTblInfo.Indices, fkInfo.RefCols...) == nil {
return infoschema.ErrForeignKeyNoIndexInParent.GenWithStackByArgs(fkInfo.Name, fkInfo.RefTable)
}
return nil
Expand Down Expand Up @@ -660,7 +660,7 @@ func checkAddForeignKeyValidInOwner(d *ddlCtx, t *meta.Meta, schema string, tbIn
return nil
}
}
if model.FindIndexByColumns(tbInfo, fk.Cols...) == nil {
if model.FindIndexByColumns(tbInfo, tbInfo.Indices, fk.Cols...) == nil {
return errors.Errorf("Failed to add the foreign key constraint. Missing index for '%s' foreign key columns in the table '%s'", fk.Name, tbInfo.Name)
}
return nil
Expand Down
7 changes: 6 additions & 1 deletion ddl/index.go
Original file line number Diff line number Diff line change
Expand Up @@ -81,7 +81,12 @@ func buildIndexColumns(ctx sessionctx.Context, columns []*model.ColumnInfo, inde
if err := checkIndexColumn(ctx, col, ip.Length); err != nil {
return nil, false, err
}
mvIndex = mvIndex || col.FieldType.IsArray()
if col.FieldType.IsArray() {
if mvIndex {
return nil, false, dbterror.ErrNotSupportedYet.GenWithStack("'more than one multi-valued key part per index'")
}
mvIndex = true
}
indexColLen := ip.Length
indexColumnLength, err := getIndexColumnLength(col, ip.Length)
if err != nil {
Expand Down
17 changes: 11 additions & 6 deletions ddl/index_cop.go
Original file line number Diff line number Diff line change
Expand Up @@ -103,9 +103,9 @@ type copReqSenderPool struct {
resultsCh chan idxRecResult
results generic.SyncMap[int, struct{}]

ctx context.Context
copCtx *copContext
startTS uint64
ctx context.Context
copCtx *copContext
store kv.Storage

senders []*copReqSender
wg sync.WaitGroup
Expand Down Expand Up @@ -139,7 +139,12 @@ func (c *copReqSender) run() {
curTaskID = task.id
logutil.BgLogger().Info("[ddl-ingest] start a cop-request task",
zap.Int("id", task.id), zap.String("task", task.String()))
rs, err := p.copCtx.buildTableScan(p.ctx, p.startTS, task.startKey, task.excludedEndKey())
ver, err := p.store.CurrentVersion(kv.GlobalTxnScope)
if err != nil {
p.resultsCh <- idxRecResult{id: task.id, err: err}
return
}
rs, err := p.copCtx.buildTableScan(p.ctx, ver.Ver, task.startKey, task.excludedEndKey())
if err != nil {
p.resultsCh <- idxRecResult{id: task.id, err: err}
return
Expand Down Expand Up @@ -167,7 +172,7 @@ func (c *copReqSender) run() {
}
}

func newCopReqSenderPool(ctx context.Context, copCtx *copContext, startTS uint64) *copReqSenderPool {
func newCopReqSenderPool(ctx context.Context, copCtx *copContext, store kv.Storage) *copReqSenderPool {
poolSize := copReadChunkPoolSize()
idxBufPool := make(chan []*indexRecord, poolSize)
srcChkPool := make(chan *chunk.Chunk, poolSize)
Expand All @@ -181,7 +186,7 @@ func newCopReqSenderPool(ctx context.Context, copCtx *copContext, startTS uint64
results: generic.NewSyncMap[int, struct{}](10),
ctx: ctx,
copCtx: copCtx,
startTS: startTS,
store: store,
senders: make([]*copReqSender, 0, variable.GetDDLReorgWorkerCounter()),
wg: sync.WaitGroup{},
idxBufPool: idxBufPool,
Expand Down
2 changes: 1 addition & 1 deletion ddl/index_cop_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -43,7 +43,7 @@ func TestAddIndexFetchRowsFromCoprocessor(t *testing.T) {
endKey := startKey.PrefixNext()
txn, err := store.Begin()
require.NoError(t, err)
idxRec, done, err := ddl.FetchRowsFromCop4Test(copCtx, startKey, endKey, txn.StartTS(), 10)
idxRec, done, err := ddl.FetchRowsFromCop4Test(copCtx, startKey, endKey, store, 10)
require.NoError(t, err)
require.False(t, done)
require.NoError(t, txn.Rollback())
Expand Down
36 changes: 35 additions & 1 deletion ddl/multi_schema_change.go
Original file line number Diff line number Diff line change
Expand Up @@ -261,7 +261,10 @@ func fillMultiSchemaInfo(info *model.MultiSchemaInfo, job *model.Job) (err error
case model.ActionRebaseAutoID, model.ActionModifyTableComment, model.ActionModifyTableCharsetAndCollate:
case model.ActionAddForeignKey:
fkInfo := job.Args[0].(*model.FKInfo)
info.ForeignKeys = append(info.ForeignKeys, fkInfo.Name)
info.AddForeignKeys = append(info.AddForeignKeys, model.AddForeignKeyInfo{
Name: fkInfo.Name,
Cols: fkInfo.Cols,
})
default:
return dbterror.ErrRunMultiSchemaChanges.FastGenByArgs(job.Type.String())
}
Expand Down Expand Up @@ -323,6 +326,32 @@ func checkOperateSameColAndIdx(info *model.MultiSchemaInfo) error {
return checkIndexes(info.AlterIndexes, true)
}

func checkOperateDropIndexUseByForeignKey(info *model.MultiSchemaInfo, t table.Table) error {
var remainIndexes, droppingIndexes []*model.IndexInfo
tbInfo := t.Meta()
for _, idx := range tbInfo.Indices {
dropping := false
for _, name := range info.DropIndexes {
if name.L == idx.Name.L {
dropping = true
break
}
}
if dropping {
droppingIndexes = append(droppingIndexes, idx)
} else {
remainIndexes = append(remainIndexes, idx)
}
}

for _, fk := range info.AddForeignKeys {
if droppingIdx := model.FindIndexByColumns(tbInfo, droppingIndexes, fk.Cols...); droppingIdx != nil && model.FindIndexByColumns(tbInfo, remainIndexes, fk.Cols...) == nil {
return dbterror.ErrDropIndexNeededInForeignKey.GenWithStackByArgs(droppingIdx.Name)
}
}
return nil
}

func checkMultiSchemaInfo(info *model.MultiSchemaInfo, t table.Table) error {
err := checkOperateSameColAndIdx(info)
if err != nil {
Expand All @@ -334,6 +363,11 @@ func checkMultiSchemaInfo(info *model.MultiSchemaInfo, t table.Table) error {
return err
}

err = checkOperateDropIndexUseByForeignKey(info, t)
if err != nil {
return err
}

return checkAddColumnTooManyColumns(len(t.Cols()) + len(info.AddColumns) - len(info.DropColumns))
}

Expand Down
5 changes: 5 additions & 0 deletions errors.toml
Original file line number Diff line number Diff line change
Expand Up @@ -766,6 +766,11 @@ error = '''
Incorrect usage of %s and %s
'''

["ddl:1235"]
error = '''
This version of TiDB doesn't yet support '%s'
'''

["ddl:1246"]
error = '''
Converting column '%s' from %s to %s
Expand Down
36 changes: 13 additions & 23 deletions executor/admin_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -1095,9 +1095,7 @@ func TestCheckFailReport(t *testing.T) {
require.NoError(t, txn.Commit(tk.ctx))

ctx, hook := withLogHook(tk.ctx, t, "inconsistency")
_, err = tk.Exec(ctx, "admin check table admin_test")
require.Error(t, err)
require.Equal(t, "[admin:8223]data inconsistency in table: admin_test, index: uk1, handle: 1, index-values:\"\" != record-values:\"handle: 1, values: [KindInt64 1]\"", err.Error())
tk.MustGetErrMsg(ctx, "admin check table admin_test", "[admin:8223]data inconsistency in table: admin_test, index: uk1, handle: 1, index-values:\"\" != record-values:\"handle: 1, values: [KindInt64 1]\"")
hook.checkLogCount(t, 1)
hook.logs[0].checkMsg(t, "admin check found data inconsistency")
hook.logs[0].checkField(t,
Expand All @@ -1119,9 +1117,7 @@ func TestCheckFailReport(t *testing.T) {
require.NoError(t, txn.Commit(tk.ctx))

ctx, hook := withLogHook(tk.ctx, t, "inconsistency")
_, err = tk.Exec(ctx, "admin check table admin_test")
require.Error(t, err)
require.Equal(t, "[admin:8223]data inconsistency in table: admin_test, index: k2, handle: 1, index-values:\"\" != record-values:\"handle: 1, values: [KindString 10]\"", err.Error())
tk.MustGetErrMsg(ctx, "admin check table admin_test", "[admin:8223]data inconsistency in table: admin_test, index: k2, handle: 1, index-values:\"\" != record-values:\"handle: 1, values: [KindString 10]\"")
hook.checkLogCount(t, 1)
hook.logs[0].checkMsg(t, "admin check found data inconsistency")
hook.logs[0].checkField(t,
Expand All @@ -1143,9 +1139,8 @@ func TestCheckFailReport(t *testing.T) {
require.NoError(t, txn.Commit(tk.ctx))

ctx, hook := withLogHook(tk.ctx, t, "inconsistency")
_, err = tk.Exec(ctx, "admin check table admin_test")
require.Error(t, err)
require.Equal(t, "[admin:8223]data inconsistency in table: admin_test, index: k2, handle: 1, index-values:\"handle: 1, values: [KindString 100 KindInt64 1]\" != record-values:\"\"", err.Error())
tk.MustGetErrMsg(ctx, "admin check table admin_test",
"[admin:8223]data inconsistency in table: admin_test, index: k2, handle: 1, index-values:\"handle: 1, values: [KindString 100 KindInt64 1]\" != record-values:\"\"")
hook.checkLogCount(t, 1)
logEntry := hook.logs[0]
logEntry.checkMsg(t, "admin check found data inconsistency")
Expand Down Expand Up @@ -1188,9 +1183,8 @@ func TestCheckFailReport(t *testing.T) {
require.NoError(t, txn.Commit(tk.ctx))

ctx, hook := withLogHook(tk.ctx, t, "inconsistency")
_, err = tk.Exec(ctx, "admin check table admin_test")
require.Error(t, err)
require.Equal(t, "[admin:8223]data inconsistency in table: admin_test, index: uk1, handle: 1, index-values:\"handle: 1, values: [KindInt64 10 KindInt64 1]\" != record-values:\"\"", err.Error())
tk.MustGetErrMsg(ctx, "admin check table admin_test",
"[admin:8223]data inconsistency in table: admin_test, index: uk1, handle: 1, index-values:\"handle: 1, values: [KindInt64 10 KindInt64 1]\" != record-values:\"\"")
hook.checkLogCount(t, 1)
logEntry := hook.logs[0]
logEntry.checkMsg(t, "admin check found data inconsistency")
Expand Down Expand Up @@ -1233,9 +1227,8 @@ func TestCheckFailReport(t *testing.T) {
require.NoError(t, err)
require.NoError(t, txn.Commit(tk.ctx))
ctx, hook := withLogHook(tk.ctx, t, "inconsistency")
_, err = tk.Exec(ctx, "admin check table admin_test")
require.Error(t, err)
require.Equal(t, "[executor:8134]data inconsistency in table: admin_test, index: uk1, col: c2, handle: \"1\", index-values:\"KindInt64 20\" != record-values:\"KindInt64 10\", compare err:<nil>", err.Error())
tk.MustGetErrMsg(ctx, "admin check table admin_test",
"[executor:8134]data inconsistency in table: admin_test, index: uk1, col: c2, handle: \"1\", index-values:\"KindInt64 20\" != record-values:\"KindInt64 10\", compare err:<nil>")
hook.checkLogCount(t, 1)
logEntry := hook.logs[0]
logEntry.checkMsg(t, "admin check found data inconsistency")
Expand All @@ -1261,9 +1254,8 @@ func TestCheckFailReport(t *testing.T) {
require.NoError(t, err)
require.NoError(t, txn.Commit(tk.ctx))
ctx, hook := withLogHook(tk.ctx, t, "inconsistency")
_, err = tk.Exec(ctx, "admin check table admin_test")
require.Error(t, err)
require.Equal(t, "[executor:8134]data inconsistency in table: admin_test, index: k2, col: c3, handle: \"1\", index-values:\"KindString 200\" != record-values:\"KindString 100\", compare err:<nil>", err.Error())
tk.MustGetErrMsg(ctx, "admin check table admin_test",
"[executor:8134]data inconsistency in table: admin_test, index: k2, col: c3, handle: \"1\", index-values:\"KindString 200\" != record-values:\"KindString 100\", compare err:<nil>")
hook.checkLogCount(t, 1)
logEntry := hook.logs[0]
logEntry.checkMsg(t, "admin check found data inconsistency")
Expand Down Expand Up @@ -1301,12 +1293,10 @@ func TestCheckFailReport(t *testing.T) {

// TODO(tiancaiamao): admin check doesn't support the chunk protocol.
// Remove this after https://github.com/pingcap/tidb/issues/35156
_, err = tk.Exec(ctx, "set @@tidb_enable_chunk_rpc = off")
require.NoError(t, err)
tk.MustExec(ctx, "set @@tidb_enable_chunk_rpc = off")

_, err = tk.Exec(ctx, "admin check table admin_test")
require.Error(t, err)
require.Equal(t, `[admin:8223]data inconsistency in table: admin_test, index: uk1, handle: 282574488403969, index-values:"handle: 282574488403969, values: [KindInt64 282578800083201 KindInt64 282574488403969]" != record-values:""`, err.Error())
tk.MustGetErrMsg(ctx, "admin check table admin_test",
`[admin:8223]data inconsistency in table: admin_test, index: uk1, handle: 282574488403969, index-values:"handle: 282574488403969, values: [KindInt64 282578800083201 KindInt64 282574488403969]" != record-values:""`)
hook.checkLogCount(t, 1)
logEntry := hook.logs[0]
logEntry.checkMsg(t, "admin check found data inconsistency")
Expand Down
11 changes: 11 additions & 0 deletions executor/bind.go
Original file line number Diff line number Diff line change
Expand Up @@ -63,6 +63,8 @@ func (e *SQLBindExec) Next(ctx context.Context, req *chunk.Chunk) error {
return e.reloadBindings()
case plannercore.OpSetBindingStatus:
return e.setBindingStatus()
case plannercore.OpSetBindingStatusByDigest:
return e.setBindingStatusByDigest()
default:
return errors.Errorf("unsupported SQL bind operation: %v", e.sqlBindOp)
}
Expand Down Expand Up @@ -119,6 +121,15 @@ func (e *SQLBindExec) setBindingStatus() error {
return err
}

func (e *SQLBindExec) setBindingStatusByDigest() error {
ok, err := domain.GetDomain(e.ctx).BindHandle().SetBindRecordStatusByDigest(e.newStatus, e.sqlDigest)
if err == nil && !ok {
warningMess := errors.New("There are no bindings can be set the status. Please check the SQL text")
e.ctx.GetSessionVars().StmtCtx.AppendWarning(warningMess)
}
return err
}

func (e *SQLBindExec) createSQLBind() error {
// For audit log, SQLBindExec execute "explain" statement internally, save and recover stmtctx
// is necessary to avoid 'create binding' been recorded as 'explain'.
Expand Down
2 changes: 1 addition & 1 deletion executor/executor.go
Original file line number Diff line number Diff line change
Expand Up @@ -1933,7 +1933,7 @@ func (e *UnionExec) Close() error {
func ResetContextOfStmt(ctx sessionctx.Context, s ast.StmtNode) (err error) {
vars := ctx.GetSessionVars()
var sc *stmtctx.StatementContext
if vars.TxnCtx.CouldRetry {
if vars.TxnCtx.CouldRetry || mysql.HasCursorExistsFlag(vars.Status) {
// Must construct new statement context object, the retry history need context for every statement.
// TODO: Maybe one day we can get rid of transaction retry, then this logic can be deleted.
sc = &stmtctx.StatementContext{}
Expand Down

0 comments on commit 3d118db

Please sign in to comment.