Skip to content

Commit

Permalink
executor: sync deletable columns to binlog when remove record (#53617) (
Browse files Browse the repository at this point in the history
#53700)

close #53133
  • Loading branch information
ti-chi-bot authored Jun 11, 2024
1 parent 236b1b7 commit 956e83d
Show file tree
Hide file tree
Showing 3 changed files with 63 additions and 1 deletion.
12 changes: 12 additions & 0 deletions ddl/column.go
Original file line number Diff line number Diff line change
Expand Up @@ -169,6 +169,9 @@ func onAddColumn(d *ddlCtx, t *meta.Meta, job *model.Job) (ver int64, err error)
case model.StateWriteReorganization:
// reorganization -> public
// Adjust table column offset.
failpoint.Inject("onAddColumnStateWriteReorg", func() {
OnAddColumnStateWriteReorgForTest()
})
offset, err := LocateOffsetToMove(columnInfo.Offset, pos, tblInfo)
if err != nil {
return ver, errors.Trace(err)
Expand All @@ -190,6 +193,9 @@ func onAddColumn(d *ddlCtx, t *meta.Meta, job *model.Job) (ver int64, err error)
return ver, errors.Trace(err)
}

// OnAddColumnStateWriteReorgForTest is only used for test.
var OnAddColumnStateWriteReorgForTest func()

// CheckAfterPositionExists makes sure the column specified in AFTER clause is exists.
// For example, ALTER TABLE t ADD COLUMN c3 INT AFTER c1.
func CheckAfterPositionExists(tblInfo *model.TableInfo, pos *ast.ColumnPosition) error {
Expand Down Expand Up @@ -263,6 +269,9 @@ func onDropColumn(d *ddlCtx, t *meta.Meta, job *model.Job) (ver int64, _ error)
}
case model.StateWriteOnly:
// write only -> delete only
failpoint.Inject("onDropColumnStateWriteOnly", func() {
OnDropColumnStateWriteOnlyForTest()
})
colInfo.State = model.StateDeleteOnly
tblInfo.MoveColumnInfo(colInfo.Offset, len(tblInfo.Columns)-1)
if len(idxInfos) > 0 {
Expand Down Expand Up @@ -313,6 +322,9 @@ func onDropColumn(d *ddlCtx, t *meta.Meta, job *model.Job) (ver int64, _ error)
return ver, errors.Trace(err)
}

// OnDropColumnStateWriteOnlyForTest is only used for test.
var OnDropColumnStateWriteOnlyForTest func()

func checkDropColumn(d *ddlCtx, t *meta.Meta, job *model.Job) (*model.TableInfo, *model.ColumnInfo, []*model.IndexInfo, bool /* ifExists */, error) {
schemaID := job.SchemaID
tblInfo, err := GetTableInfoAndCancelFaultJob(t, job, schemaID)
Expand Down
49 changes: 49 additions & 0 deletions executor/executor_txn_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -19,9 +19,13 @@ import (
"fmt"
"strconv"
"strings"
"sync"
"testing"
"time"

"github.com/pingcap/failpoint"
"github.com/pingcap/tidb/ddl"
"github.com/pingcap/tidb/errno"
"github.com/pingcap/tidb/executor"
"github.com/pingcap/tidb/sessionctx/binloginfo"
"github.com/pingcap/tidb/testkit"
Expand Down Expand Up @@ -798,3 +802,48 @@ func (m mockPumpClient) WriteBinlog(ctx context.Context, in *binlog.WriteBinlogR
func (m mockPumpClient) PullBinlogs(ctx context.Context, in *binlog.PullBinlogReq, opts ...grpc.CallOption) (binlog.Pump_PullBinlogsClient, error) {
return nil, nil
}

func TestColumnNotMatchError(t *testing.T) {
store := testkit.CreateMockStore(t)
tk := testkit.NewTestKit(t, store)
tk.Session().GetSessionVars().BinlogClient = binloginfo.MockPumpsClient(&mockPumpClient{})
tk.MustExec("set @@global.tidb_enable_metadata_lock=0")
tk.MustExec("use test")
tk2 := testkit.NewTestKit(t, store)
tk2.MustExec("use test")
tk.MustExec("create table t(id int primary key, a int)")
tk.MustExec("insert into t values(1, 2)")

ddl.OnAddColumnStateWriteReorgForTest = func() {
tk.MustExec("begin;")
}
require.NoError(t, failpoint.Enable("github.com/pingcap/tidb/ddl/onAddColumnStateWriteReorg", "return"))
defer func() {
require.NoError(t, failpoint.Disable("github.com/pingcap/tidb/ddl/onAddColumnStateWriteReorg"))
}()
var wg sync.WaitGroup
wg.Add(1)
go func() {
tk2.MustExec("alter table t add column wait_notify int")
wg.Done()
}()
wg.Wait()
tk.MustExec("delete from t where id=1")
tk.MustGetErrCode("commit", errno.ErrInfoSchemaChanged)

ddl.OnDropColumnStateWriteOnlyForTest = func() {
tk.MustExec("begin;")
}
require.NoError(t, failpoint.Enable("github.com/pingcap/tidb/ddl/onDropColumnStateWriteOnly", "return"))
defer func() {
require.NoError(t, failpoint.Disable("github.com/pingcap/tidb/ddl/onDropColumnStateWriteOnly"))
}()
wg.Add(1)
go func() {
tk2.MustExec("alter table t drop column wait_notify")
wg.Done()
}()
wg.Wait()
tk.MustExec("delete from t where id=1")
tk.MustGetErrCode("commit", errno.ErrInfoSchemaChanged)
}
3 changes: 2 additions & 1 deletion table/tables/tables.go
Original file line number Diff line number Diff line change
Expand Up @@ -1175,7 +1175,8 @@ func (t *TableCommon) RemoveRecord(ctx sessionctx.Context, h kv.Handle, r []type
memBuffer.Release(sh)

if shouldWriteBinlog(ctx, t.meta) {
cols := t.Cols()
cols := t.DeletableCols()

colIDs := make([]int64, 0, len(cols)+1)
for _, col := range cols {
colIDs = append(colIDs, col.ID)
Expand Down

0 comments on commit 956e83d

Please sign in to comment.