Skip to content

Commit 0a3f34c

Browse files
Defined2014RidRisR
authored andcommitted
*: remove ExtraPidCol and replace it with ExtraPhysTblIDCol (#53974)
close #53929
1 parent fd21de7 commit 0a3f34c

29 files changed

+121
-135
lines changed

pkg/executor/builder.go

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -683,7 +683,7 @@ func (b *executorBuilder) buildCleanupIndex(v *plannercore.CleanupIndex) exec.Ex
683683
sessCtx := e.Ctx().GetSessionVars().StmtCtx
684684
e.handleCols = buildHandleColsForExec(sessCtx, tblInfo, e.columns)
685685
if e.index.Meta().Global {
686-
e.columns = append(e.columns, model.NewExtraPartitionIDColInfo())
686+
e.columns = append(e.columns, model.NewExtraPhysTblIDColInfo())
687687
}
688688
return e
689689
}
@@ -5133,7 +5133,7 @@ func NewRowDecoder(ctx sessionctx.Context, schema *expression.Schema, tbl *model
51335133
}
51345134
defVal := func(i int, chk *chunk.Chunk) error {
51355135
if reqCols[i].ID < 0 {
5136-
// model.ExtraHandleID, ExtraPidColID, ExtraPhysTblID... etc
5136+
// model.ExtraHandleID, ExtraPhysTblID... etc
51375137
// Don't set the default value for that column.
51385138
chk.AppendNull(i)
51395139
return nil

pkg/executor/delete.go

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -130,7 +130,7 @@ func (e *DeleteExec) deleteSingleTableByChunk(ctx context.Context) error {
130130

131131
datumRow := make([]types.Datum, 0, len(fields))
132132
for i, field := range fields {
133-
if columns[i].ID == model.ExtraPidColID || columns[i].ID == model.ExtraPhysTblID {
133+
if columns[i].ID == model.ExtraPhysTblID {
134134
continue
135135
}
136136

pkg/executor/distsql.go

Lines changed: 2 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -581,24 +581,22 @@ func (e *IndexLookUpExecutor) startWorkers(ctx context.Context, initBatchSize in
581581

582582
func (e *IndexLookUpExecutor) needPartitionHandle(tp getHandleType) (bool, error) {
583583
var col *expression.Column
584-
var needPartitionHandle, hasExtraCol bool
584+
var needPartitionHandle bool
585585
if tp == getHandleFromIndex {
586586
cols := e.idxPlans[0].Schema().Columns
587587
outputOffsets := e.dagPB.OutputOffsets
588588
col = cols[outputOffsets[len(outputOffsets)-1]]
589589
// For indexScan, need partitionHandle when global index or keepOrder with partitionTable
590590
needPartitionHandle = e.index.Global || e.partitionTableMode && e.keepOrder
591-
hasExtraCol = col.ID == model.ExtraPhysTblID || col.ID == model.ExtraPidColID
592591
} else {
593592
cols := e.tblPlans[0].Schema().Columns
594593
outputOffsets := e.tableRequest.OutputOffsets
595594
col = cols[outputOffsets[len(outputOffsets)-1]]
596595

597596
// For TableScan, need partitionHandle in `indexOrder` when e.keepOrder == true or execute `admin check [table|index]` with global index
598597
needPartitionHandle = ((e.index.Global || e.partitionTableMode) && e.keepOrder) || (e.index.Global && e.checkIndexValue != nil)
599-
// no ExtraPidColID here, because TableScan shouldn't contain them.
600-
hasExtraCol = col.ID == model.ExtraPhysTblID
601598
}
599+
hasExtraCol := col.ID == model.ExtraPhysTblID
602600

603601
// There will be two needPartitionHandle != hasExtraCol situations.
604602
// Only `needPartitionHandle` == true and `hasExtraCol` == false are not allowed.

pkg/executor/index_merge_reader.go

Lines changed: 5 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -628,7 +628,6 @@ func (w *partialTableWorker) needPartitionHandle() (bool, error) {
628628
col := cols[outputOffsets[len(outputOffsets)-1]]
629629

630630
needPartitionHandle := w.partitionTableMode && len(w.byItems) > 0
631-
// no ExtraPidColID here, because a clustered index couldn't be a global index.
632631
hasExtraCol := col.ID == model.ExtraPhysTblID
633632

634633
// There will be two needPartitionHandle != hasExtraCol situations.
@@ -1698,24 +1697,25 @@ func syncErr(ctx context.Context, finished <-chan struct{}, errCh chan<- *indexM
16981697
}
16991698

17001699
// needPartitionHandle indicates whether we need create a partitionHandle or not.
1701-
// If the schema from planner part contains ExtraPidColID or ExtraPhysTblID,
1700+
// If the schema from planner part contains ExtraPhysTblID,
17021701
// we need create a partitionHandle, otherwise create a normal handle.
17031702
// In TableRowIDScan, the partitionHandle will be used to create key ranges.
17041703
func (w *partialIndexWorker) needPartitionHandle() (bool, error) {
17051704
cols := w.plan[0].Schema().Columns
17061705
outputOffsets := w.dagPB.OutputOffsets
17071706
col := cols[outputOffsets[len(outputOffsets)-1]]
17081707

1709-
needPartitionHandle := w.partitionTableMode && len(w.byItems) > 0
1710-
hasExtraCol := col.ID == model.ExtraPidColID || col.ID == model.ExtraPhysTblID
1708+
is := w.plan[0].(*plannercore.PhysicalIndexScan)
1709+
needPartitionHandle := w.partitionTableMode && len(w.byItems) > 0 || is.Index.Global
1710+
hasExtraCol := col.ID == model.ExtraPhysTblID
17111711

17121712
// There will be two needPartitionHandle != hasExtraCol situations.
17131713
// Only `needPartitionHandle` == true and `hasExtraCol` == false are not allowed.
17141714
// `ExtraPhysTblID` will be used in `SelectLock` when `needPartitionHandle` == false and `hasExtraCol` == true.
17151715
if needPartitionHandle && !hasExtraCol {
17161716
return needPartitionHandle, errors.Errorf("Internal error, needPartitionHandle != ret")
17171717
}
1718-
return needPartitionHandle || (col.ID == model.ExtraPidColID), nil
1718+
return needPartitionHandle, nil
17191719
}
17201720

17211721
func (w *partialIndexWorker) fetchHandles(

pkg/executor/partition_table_test.go

Lines changed: 29 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -2407,3 +2407,32 @@ func TestIssue31024(t *testing.T) {
24072407

24082408
tk2.MustExec("rollback")
24092409
}
2410+
2411+
func TestGlobalIndexWithSelectLock(t *testing.T) {
2412+
store := testkit.CreateMockStore(t)
2413+
2414+
tk1 := testkit.NewTestKit(t, store)
2415+
tk1.MustExec("set tidb_enable_global_index = true")
2416+
tk1.MustExec("use test")
2417+
tk1.MustExec("create table t(a int, b int, unique index(b), primary key(a)) partition by hash(a) partitions 5;")
2418+
tk1.MustExec("insert into t values (1,1),(2,2),(3,3),(4,4),(5,5);")
2419+
tk1.MustExec("begin")
2420+
tk1.MustExec("select * from t use index(b) where b = 2 order by b limit 1 for update;")
2421+
2422+
tk2 := testkit.NewTestKit(t, store)
2423+
tk2.MustExec("use test")
2424+
2425+
ch := make(chan int, 10)
2426+
go func() {
2427+
// Check the key is locked.
2428+
tk2.MustExec("update t set b = 6 where b = 2")
2429+
ch <- 1
2430+
}()
2431+
2432+
time.Sleep(50 * time.Millisecond)
2433+
ch <- 0
2434+
tk1.MustExec("commit")
2435+
2436+
require.Equal(t, <-ch, 0)
2437+
require.Equal(t, <-ch, 1)
2438+
}

pkg/expression/explain.go

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -37,11 +37,11 @@ func (expr *ScalarFunction) explainInfo(ctx EvalContext, normalized bool) string
3737
intest.Assert(normalized || ctx != nil)
3838
var buffer bytes.Buffer
3939
fmt.Fprintf(&buffer, "%s(", expr.FuncName.L)
40-
// convert `in(_tidb_pid, -1)` to `in(_tidb_pid, dual)` whether normalized equals to true or false.
40+
// convert `in(_tidb_tid, -1)` to `in(_tidb_tid, dual)` whether normalized equals to true or false.
4141
if expr.FuncName.L == ast.In {
4242
args := expr.GetArgs()
43-
if len(args) == 2 && args[0].ExplainNormalizedInfo() == model.ExtraPartitionIdName.L && args[1].(*Constant).Value.GetInt64() == -1 {
44-
buffer.WriteString(model.ExtraPartitionIdName.L + ", dual)")
43+
if len(args) == 2 && strings.HasSuffix(args[0].ExplainNormalizedInfo(), model.ExtraPhysTblIdName.L) && args[1].(*Constant).Value.GetInt64() == -1 {
44+
buffer.WriteString(args[0].ExplainNormalizedInfo() + ", dual)")
4545
return buffer.String()
4646
}
4747
}

pkg/parser/model/model.go

Lines changed: 5 additions & 21 deletions
Original file line numberDiff line numberDiff line change
@@ -383,13 +383,12 @@ func IsIndexPrefixCovered(tbInfo *TableInfo, index *IndexInfo, cols ...CIStr) bo
383383
// for use of execution phase.
384384
const ExtraHandleID = -1
385385

386-
// ExtraPidColID is the column ID of column which store the partitionID decoded in global index values.
387-
const ExtraPidColID = -2
386+
// Deprecated: Use ExtraPhysTblID instead.
387+
// const ExtraPidColID = -2
388388

389389
// ExtraPhysTblID is the column ID of column that should be filled in with the physical table id.
390390
// Primarily used for table partition dynamic prune mode, to return which partition (physical table id) the row came from.
391-
// Using a dedicated id for this, since in the future ExtraPidColID and ExtraPhysTblID may be used for the same request.
392-
// Must be after ExtraPidColID!
391+
// If used with a global index, the partition ID decoded from the key value will be filled in.
393392
const ExtraPhysTblID = -3
394393

395394
// ExtraRowChecksumID is the column ID of column which holds the row checksum info.
@@ -435,8 +434,8 @@ const (
435434
// ExtraHandleName is the name of ExtraHandle Column.
436435
var ExtraHandleName = NewCIStr("_tidb_rowid")
437436

438-
// ExtraPartitionIdName is the name of ExtraPartitionId Column.
439-
var ExtraPartitionIdName = NewCIStr("_tidb_pid") //nolint:revive
437+
// Deprecated: Use ExtraPhysTblIdName instead.
438+
// var ExtraPartitionIdName = NewCIStr("_tidb_pid") //nolint:revive
440439

441440
// ExtraPhysTblIdName is the name of ExtraPhysTblID Column.
442441
var ExtraPhysTblIdName = NewCIStr("_tidb_tid") //nolint:revive
@@ -923,21 +922,6 @@ func NewExtraHandleColInfo() *ColumnInfo {
923922
return colInfo
924923
}
925924

926-
// NewExtraPartitionIDColInfo mocks a column info for extra partition id column.
927-
func NewExtraPartitionIDColInfo() *ColumnInfo {
928-
colInfo := &ColumnInfo{
929-
ID: ExtraPidColID,
930-
Name: ExtraPartitionIdName,
931-
}
932-
colInfo.SetType(mysql.TypeLonglong)
933-
flen, decimal := mysql.GetDefaultFieldLengthAndDecimal(mysql.TypeLonglong)
934-
colInfo.SetFlen(flen)
935-
colInfo.SetDecimal(decimal)
936-
colInfo.SetCharset(charset.CharsetBin)
937-
colInfo.SetCollate(charset.CollationBin)
938-
return colInfo
939-
}
940-
941925
// NewExtraPhysTblIDColInfo mocks a column info for extra partition id column.
942926
func NewExtraPhysTblIDColInfo() *ColumnInfo {
943927
colInfo := &ColumnInfo{

pkg/planner/core/find_best_task.go

Lines changed: 17 additions & 21 deletions
Original file line numberDiff line numberDiff line change
@@ -1423,15 +1423,6 @@ func (ds *DataSource) FindBestTask(prop *property.PhysicalProperty, planCounter
14231423
}
14241424
}
14251425
if canConvertPointGet {
1426-
// If the schema contains ExtraPidColID, do not convert to point get.
1427-
// Because the point get executor can not handle the extra partition ID column now.
1428-
// I.e. Global Index is used
1429-
for _, col := range ds.schema.Columns {
1430-
if col.ID == model.ExtraPidColID {
1431-
canConvertPointGet = false
1432-
break
1433-
}
1434-
}
14351426
if path != nil && path.Index != nil && path.Index.Global {
14361427
// Don't convert to point get during ddl
14371428
// TODO: Revisit truncate partition and global index
@@ -2148,6 +2139,15 @@ func (is *PhysicalIndexScan) initSchema(idxExprCols []*expression.Column, isDoub
21482139
}
21492140
}
21502141

2142+
var extraPhysTblCol *expression.Column
2143+
// If `dataSouceSchema` contains `model.ExtraPhysTblID`, we should add it into `indexScan.schema`
2144+
for _, col := range is.dataSourceSchema.Columns {
2145+
if col.ID == model.ExtraPhysTblID {
2146+
extraPhysTblCol = col.Clone().(*expression.Column)
2147+
break
2148+
}
2149+
}
2150+
21512151
if isDoubleRead || is.Index.Global {
21522152
// If it's double read case, the first index must return handle. So we should add extra handle column
21532153
// if there isn't a handle column.
@@ -2161,23 +2161,19 @@ func (is *PhysicalIndexScan) initSchema(idxExprCols []*expression.Column, isDoub
21612161
})
21622162
}
21632163
}
2164-
// If it's global index, handle and PidColID columns has to be added, so that needed pids can be filtered.
2165-
if is.Index.Global {
2164+
// If it's global index, handle and PhysTblID columns has to be added, so that needed pids can be filtered.
2165+
if is.Index.Global && extraPhysTblCol == nil {
21662166
indexCols = append(indexCols, &expression.Column{
21672167
RetType: types.NewFieldType(mysql.TypeLonglong),
2168-
ID: model.ExtraPidColID,
2168+
ID: model.ExtraPhysTblID,
21692169
UniqueID: is.SCtx().GetSessionVars().AllocPlanColumnID(),
2170-
OrigName: model.ExtraPartitionIdName.O,
2170+
OrigName: model.ExtraPhysTblIdName.O,
21712171
})
21722172
}
21732173
}
21742174

2175-
// If `dataSouceSchema` contains `model.ExtraPhysTblID`, we should add it into `indexScan.schema`
2176-
for _, col := range is.dataSourceSchema.Columns {
2177-
if col.ID == model.ExtraPhysTblID {
2178-
indexCols = append(indexCols, col.Clone().(*expression.Column))
2179-
break
2180-
}
2175+
if extraPhysTblCol != nil {
2176+
indexCols = append(indexCols, extraPhysTblCol)
21812177
}
21822178

21832179
is.SetSchema(expression.NewSchema(indexCols...))
@@ -2189,14 +2185,14 @@ func (is *PhysicalIndexScan) addSelectionConditionForGlobalIndex(p *DataSource,
21892185
}
21902186
args := make([]expression.Expression, 0, len(p.partitionNames)+1)
21912187
for _, col := range is.schema.Columns {
2192-
if col.ID == model.ExtraPidColID {
2188+
if col.ID == model.ExtraPhysTblID {
21932189
args = append(args, col.Clone())
21942190
break
21952191
}
21962192
}
21972193

21982194
if len(args) != 1 {
2199-
return nil, errors.Errorf("Can't find column %s in schema %s", model.ExtraPartitionIdName.O, is.schema)
2195+
return nil, errors.Errorf("Can't find column %s in schema %s", model.ExtraPhysTblIdName.O, is.schema)
22002196
}
22012197

22022198
// For SQL like 'select x from t partition(p0, p1) use index(idx)',

pkg/planner/core/logical_plan_builder.go

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -3892,7 +3892,7 @@ func unfoldWildStar(field *ast.SelectField, outputName types.NameSlice, column [
38923892
}
38933893
if (dbName.L == "" || dbName.L == name.DBName.L) &&
38943894
(tblName.L == "" || tblName.L == name.TblName.L) &&
3895-
col.ID != model.ExtraHandleID && col.ID != model.ExtraPidColID && col.ID != model.ExtraPhysTblID {
3895+
col.ID != model.ExtraHandleID && col.ID != model.ExtraPhysTblID {
38963896
colName := &ast.ColumnNameExpr{
38973897
Name: &ast.ColumnName{
38983898
Schema: name.DBName,

pkg/planner/core/plan_to_pb.go

Lines changed: 0 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -465,8 +465,6 @@ func (p *PhysicalIndexScan) ToPB(_ *base.BuildPBContext, _ kv.StoreType) (*tipb.
465465
columns = append(columns, model.NewExtraHandleColInfo())
466466
} else if col.ID == model.ExtraPhysTblID {
467467
columns = append(columns, model.NewExtraPhysTblIDColInfo())
468-
} else if col.ID == model.ExtraPidColID {
469-
columns = append(columns, model.NewExtraPartitionIDColInfo())
470468
} else {
471469
columns = append(columns, FindColumnInfoByID(tableColumns, col.ID))
472470
}

0 commit comments

Comments
 (0)