Skip to content

Commit 02912b5

Browse files
craig[bot]jeffswenson
craig[bot]
andcommitted
Merge #143773
143773: logical: require matching column ids when using the kv writer r=jeffswenson a=jeffswenson Previously, when a cput failed, the source descriptor was used to decode the returned KV. That only works if the source descriptor has the same physical encoding. This is usually true for LDR, but the encodings will differ if the tables have different column ids. In 25.2 and beyond, the SQL writer will be the default implementation of LDR and we plan on deleting the KV writer in near future. In 25.1 users should be able to avoid this problem by using the `CREATE LOGICALLY REPLICATED` table syntax, which uses a copy of the source descriptor. Fixes: #143754 Release note (bug fix): validate column ids when starting an immediate mode logical replication stream. Co-authored-by: Jeff Swenson <[email protected]>
2 parents 8d3b79b + f184e6e commit 02912b5

File tree

3 files changed

+81
-67
lines changed

3 files changed

+81
-67
lines changed

Diff for: pkg/crosscluster/logical/create_logical_replication_stmt.go

+3-2
Original file line numberDiff line numberDiff line change
@@ -477,13 +477,14 @@ func doLDRPlan(
477477
}
478478
for i := range srcExternalCatalog.Tables {
479479
destTableDesc := dstTableDescs[i]
480+
mayUseKVWriter := false
480481
if details.Mode != jobspb.LogicalReplicationDetails_Validated {
481482
if len(destTableDesc.OutboundForeignKeys()) > 0 || len(destTableDesc.InboundForeignKeys()) > 0 {
482483
return pgerror.Newf(pgcode.InvalidParameterValue, "foreign keys are only supported with MODE = 'validated'")
483484
}
485+
mayUseKVWriter = true
484486
}
485-
486-
err := tabledesc.CheckLogicalReplicationCompatibility(&srcExternalCatalog.Tables[i], destTableDesc.TableDesc(), skipSchemaCheck || details.CreateTable)
487+
err := tabledesc.CheckLogicalReplicationCompatibility(&srcExternalCatalog.Tables[i], destTableDesc.TableDesc(), skipSchemaCheck || details.CreateTable, mayUseKVWriter)
487488
if err != nil {
488489
return err
489490
}

Diff for: pkg/crosscluster/logical/logical_replication_job_test.go

+64-62
Original file line numberDiff line numberDiff line change
@@ -2415,6 +2415,46 @@ func TestLogicalReplicationGatewayRoute(t *testing.T) {
24152415
require.Empty(t, progress.Details.(*jobspb.Progress_LogicalReplication).LogicalReplication.PartitionConnUris)
24162416
}
24172417

2418+
func TestMismatchColIDs(t *testing.T) {
2419+
defer leaktest.AfterTest(t)()
2420+
skip.UnderDeadlock(t)
2421+
defer log.Scope(t).Close(t)
2422+
2423+
ctx := context.Background()
2424+
tc, s, sqlA, sqlB := setupLogicalTestServer(t, ctx, testClusterBaseClusterArgs, 1)
2425+
defer tc.Stopper().Stop(ctx)
2426+
2427+
dbBURL := replicationtestutils.GetExternalConnectionURI(t, s, s, serverutils.DBName("b"))
2428+
2429+
createStmt := "CREATE TABLE foo (pk int primary key, payload string)"
2430+
sqlA.Exec(t, createStmt)
2431+
sqlA.Exec(t, "ALTER TABLE foo ADD COLUMN baz INT DEFAULT 2")
2432+
2433+
// Insert some data into foo
2434+
sqlA.Exec(t, "INSERT INTO foo VALUES (1, 'hello')")
2435+
sqlA.Exec(t, "INSERT INTO foo VALUES (2, 'world')")
2436+
2437+
sqlB.Exec(t, createStmt)
2438+
sqlB.Exec(t, "ALTER TABLE foo ADD COLUMN bar INT DEFAULT 2")
2439+
2440+
sqlB.Exec(t, "ALTER TABLE foo ADD COLUMN baz INT DEFAULT 2")
2441+
sqlB.Exec(t, "INSERT INTO foo VALUES (3, 'hello', 3)")
2442+
sqlB.Exec(t, "ALTER TABLE foo DROP COLUMN bar")
2443+
sqlB.Exec(t, "INSERT INTO foo VALUES (4, 'world')")
2444+
2445+
// LDR immediate mode creation should fail because of mismatched column IDs.
2446+
sqlA.ExpectErr(t,
2447+
"destination table foo column baz has ID 3, but the source table foo has ID 4",
2448+
"CREATE LOGICAL REPLICATION STREAM FROM TABLE foo ON $1 INTO TABLE foo WITH MODE = 'immediate'", dbBURL.String())
2449+
2450+
// LDR validated mode creation should succeed because the SQL writer supports mismatched column IDs.
2451+
var jobID jobspb.JobID
2452+
sqlA.QueryRow(t, "CREATE LOGICAL REPLICATION STREAM FROM TABLE foo ON $1 INTO TABLE foo WITH MODE = 'validated'", dbBURL.String()).Scan(&jobID)
2453+
2454+
now := s.Clock().Now()
2455+
WaitUntilReplicatedTime(t, now, sqlA, jobID)
2456+
}
2457+
24182458
// TestLogicalReplicationCreationChecks verifies that we check that the table
24192459
// schemas are compatible when creating the replication stream.
24202460
func TestLogicalReplicationCreationChecks(t *testing.T) {
@@ -2437,40 +2477,33 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
24372477

24382478
dbBURL := replicationtestutils.GetExternalConnectionURI(t, s, s, serverutils.DBName("b"))
24392479

2480+
expectErr := func(t *testing.T, tableName string, err string) {
2481+
t.Helper()
2482+
dbA.ExpectErr(t, err, fmt.Sprintf("CREATE LOGICAL REPLICATION STREAM FROM TABLE %s ON $1 INTO TABLE %s WITH MODE = 'validated'", tableName, tableName), dbBURL.String())
2483+
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
2484+
}
2485+
24402486
// Column families are not allowed.
24412487
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN new_col INT NOT NULL CREATE FAMILY f1")
24422488
dbB.Exec(t, "ALTER TABLE b.tab ADD COLUMN new_col INT NOT NULL")
2443-
dbA.ExpectErr(t,
2444-
"cannot create logical replication stream: table tab has more than one column family",
2445-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2446-
)
2447-
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
2489+
expectErr(t, "tab", "cannot create logical replication stream: table tab has more than one column family")
24482490

24492491
// UniqueWithoutIndex constraints are not allowed.
24502492
for _, db := range []*sqlutils.SQLRunner{dbA, dbB} {
24512493
db.Exec(t, "SET experimental_enable_unique_without_index_constraints = true")
24522494
db.Exec(t, "CREATE TABLE tab_with_uwi (pk INT PRIMARY KEY, v INT UNIQUE WITHOUT INDEX)")
24532495
}
2454-
dbA.ExpectErr(t,
2455-
"cannot create logical replication stream: table tab_with_uwi has UNIQUE WITHOUT INDEX constraints: unique_v",
2456-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab_with_uwi ON $1 INTO TABLE tab_with_uwi", dbBURL.String(),
2457-
)
2496+
expectErr(t, "tab_with_uwi", "cannot create logical replication stream: table tab_with_uwi has UNIQUE WITHOUT INDEX constraints: unique_v")
24582497
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
24592498

24602499
// Check for mismatched numbers of columns.
24612500
dbA.Exec(t, "ALTER TABLE tab DROP COLUMN new_col")
2462-
dbA.ExpectErr(t,
2463-
"cannot create logical replication stream: destination table tab has 2 columns, but the source table tab has 3 columns",
2464-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2465-
)
2501+
expectErr(t, "tab", "cannot create logical replication stream: destination table tab has 2 columns, but the source table tab has 3 columns")
24662502
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
24672503

24682504
// Check for mismatched column types.
24692505
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN new_col TEXT NOT NULL")
2470-
dbA.ExpectErr(t,
2471-
"cannot create logical replication stream: destination table tab column new_col has type STRING, but the source table tab has type INT8",
2472-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2473-
)
2506+
expectErr(t, "tab", "cannot create logical replication stream: destination table tab column new_col has type STRING, but the source table tab has type INT8")
24742507
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
24752508

24762509
// Check for composite type in primary key.
@@ -2479,39 +2512,27 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
24792512
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN composite_col DECIMAL NOT NULL")
24802513
dbB.Exec(t, "ALTER TABLE b.tab ADD COLUMN composite_col DECIMAL NOT NULL")
24812514
dbA.Exec(t, "ALTER TABLE tab ALTER PRIMARY KEY USING COLUMNS (pk, composite_col)")
2482-
dbA.ExpectErr(t,
2483-
`cannot create logical replication stream: table tab has a primary key column \(composite_col\) with composite encoding`,
2484-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2485-
)
2515+
expectErr(t, "tab", `cannot create logical replication stream: table tab has a primary key column \(composite_col\) with composite encoding`)
24862516
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
24872517

24882518
// Check for partial indexes.
24892519
dbA.Exec(t, "ALTER TABLE tab ALTER PRIMARY KEY USING COLUMNS (pk)")
24902520
dbA.Exec(t, "CREATE INDEX partial_idx ON tab(composite_col) WHERE pk > 0")
2491-
dbA.ExpectErr(t,
2492-
`cannot create logical replication stream: table tab has a partial index partial_idx`,
2493-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2494-
)
2521+
expectErr(t, "tab", "cannot create logical replication stream: table tab has a partial index partial_idx")
24952522
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
24962523

24972524
// Check for virtual computed columns that are a key of a secondary index.
24982525
dbA.Exec(t, "DROP INDEX partial_idx")
24992526
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN virtual_col INT NOT NULL AS (pk + 1) VIRTUAL")
25002527
dbB.Exec(t, "ALTER TABLE b.tab ADD COLUMN virtual_col INT NOT NULL AS (pk + 1) VIRTUAL")
25012528
dbA.Exec(t, "CREATE INDEX virtual_col_idx ON tab(virtual_col)")
2502-
dbA.ExpectErr(t,
2503-
`cannot create logical replication stream: table tab has a virtual computed column virtual_col that is a key of index virtual_col_idx`,
2504-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2505-
)
2529+
expectErr(t, "tab", "cannot create logical replication stream: table tab has a virtual computed column virtual_col that is a key of index virtual_col_idx")
25062530
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
25072531

25082532
// Check for virtual columns that are in the primary index.
25092533
dbA.Exec(t, "DROP INDEX virtual_col_idx")
25102534
dbA.Exec(t, "ALTER TABLE tab ALTER PRIMARY KEY USING COLUMNS (pk, virtual_col)")
2511-
dbA.ExpectErr(t,
2512-
`cannot create logical replication stream: table tab has a virtual computed column virtual_col that appears in the primary key`,
2513-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2514-
)
2535+
expectErr(t, "tab", "cannot create logical replication stream: table tab has a virtual computed column virtual_col that appears in the primary key")
25152536
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
25162537

25172538
// Change the primary key back, and remove the indexes that are left over from
@@ -2524,10 +2545,7 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
25242545
// Check that CHECK constraints match.
25252546
dbA.Exec(t, "ALTER TABLE tab ADD CONSTRAINT check_constraint_1 CHECK (pk > 0)")
25262547
dbB.Exec(t, "ALTER TABLE b.tab ADD CONSTRAINT check_constraint_1 CHECK (length(payload) > 1)")
2527-
dbA.ExpectErr(t,
2528-
`cannot create logical replication stream: destination table tab CHECK constraints do not match source table tab`,
2529-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2530-
)
2548+
expectErr(t, "tab", "cannot create logical replication stream: destination table tab CHECK constraints do not match source table tab")
25312549
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
25322550

25332551
// Allow user to create LDR stream with mismatched CHECK via SKIP SCHEMA CHECK.
@@ -2545,7 +2563,7 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
25452563
dbB.Exec(t, "ALTER TABLE b.tab ADD CONSTRAINT check_constraint_2 CHECK (pk > 0)")
25462564
var jobAID jobspb.JobID
25472565
dbA.QueryRow(t,
2548-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab",
2566+
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab WITH MODE = 'validated'",
25492567
dbBURL.String(),
25502568
).Scan(&jobAID)
25512569

@@ -2559,10 +2577,7 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
25592577
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN udf_col INT NOT NULL")
25602578
dbA.Exec(t, "ALTER TABLE tab ALTER COLUMN udf_col SET DEFAULT my_udf()")
25612579
dbB.Exec(t, "ALTER TABLE tab ADD COLUMN udf_col INT NOT NULL DEFAULT 1")
2562-
dbA.ExpectErr(t,
2563-
`cannot create logical replication stream: table tab references functions with IDs \[[0-9]+\]`,
2564-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2565-
)
2580+
expectErr(t, "tab", "cannot create logical replication stream: table tab references functions with IDs [[0-9]+]")
25662581
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
25672582

25682583
// Check if the table references a sequence.
@@ -2571,21 +2586,15 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
25712586
dbA.Exec(t, "CREATE SEQUENCE my_seq")
25722587
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN seq_col INT NOT NULL DEFAULT nextval('my_seq')")
25732588
dbB.Exec(t, "ALTER TABLE tab ADD COLUMN seq_col INT NOT NULL DEFAULT 1")
2574-
dbA.ExpectErr(t,
2575-
`cannot create logical replication stream: table tab references sequences with IDs \[[0-9]+\]`,
2576-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2577-
)
2589+
expectErr(t, "tab", "cannot create logical replication stream: table tab references sequences with IDs [[0-9]+]")
25782590
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
25792591

25802592
// Check if table has a trigger.
25812593
dbA.Exec(t, "ALTER TABLE tab DROP COLUMN seq_col")
25822594
dbB.Exec(t, "ALTER TABLE tab DROP COLUMN seq_col")
25832595
dbA.Exec(t, "CREATE OR REPLACE FUNCTION my_trigger() RETURNS TRIGGER AS $$ BEGIN RETURN NEW; END $$ LANGUAGE PLPGSQL")
25842596
dbA.Exec(t, "CREATE TRIGGER my_trigger BEFORE INSERT ON tab FOR EACH ROW EXECUTE FUNCTION my_trigger()")
2585-
dbA.ExpectErr(t,
2586-
`cannot create logical replication stream: table tab references triggers \[my_trigger\]`,
2587-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2588-
)
2597+
expectErr(t, "tab", `cannot create logical replication stream: table tab references triggers \[my_trigger\]`)
25892598
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
25902599

25912600
// Verify that the stream cannot be created with mismatched enum types.
@@ -2594,9 +2603,8 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
25942603
dbB.Exec(t, "CREATE TYPE b.mytype AS ENUM ('a', 'b')")
25952604
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN enum_col mytype NOT NULL")
25962605
dbB.Exec(t, "ALTER TABLE b.tab ADD COLUMN enum_col b.mytype NOT NULL")
2597-
dbA.ExpectErr(t,
2606+
expectErr(t, "tab",
25982607
`cannot create logical replication stream: .* destination type USER DEFINED ENUM: public.mytype has logical representations \[a b c\], but the source type USER DEFINED ENUM: mytype has \[a b\]`,
2599-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
26002608
)
26012609
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
26022610

@@ -2616,21 +2624,15 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
26162624
dbB.Exec(t, "CREATE TYPE b.composite_typ AS (a TEXT, b INT)")
26172625
dbA.Exec(t, "ALTER TABLE tab ADD COLUMN composite_udt_col composite_typ NOT NULL")
26182626
dbB.Exec(t, "ALTER TABLE b.tab ADD COLUMN composite_udt_col b.composite_typ NOT NULL")
2619-
dbA.ExpectErr(t,
2620-
`cannot create logical replication stream: .* destination type USER DEFINED RECORD: public.composite_typ tuple element 0 does not match source type USER DEFINED RECORD: composite_typ tuple element 0: destination type INT8 does not match source type STRING`,
2621-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2622-
)
2627+
expectErr(t, "tab", "cannot create logical replication stream: .* destination type USER DEFINED RECORD: public.composite_typ tuple element 0 does not match source type USER DEFINED RECORD: composite_typ tuple element 0: destination type INT8 does not match source type STRING")
26232628
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
26242629

26252630
// Check that UNIQUE indexes match.
26262631
dbA.Exec(t, "ALTER TABLE tab DROP COLUMN composite_udt_col")
26272632
dbB.Exec(t, "ALTER TABLE b.tab DROP COLUMN composite_udt_col")
26282633
dbA.Exec(t, "CREATE UNIQUE INDEX payload_idx ON tab(payload)")
26292634
dbB.Exec(t, "CREATE UNIQUE INDEX multi_idx ON b.tab(composite_col, pk)")
2630-
dbA.ExpectErr(t,
2631-
`cannot create logical replication stream: destination table tab UNIQUE indexes do not match source table tab`,
2632-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab", dbBURL.String(),
2633-
)
2635+
expectErr(t, "tab", "cannot create logical replication stream: destination table tab UNIQUE indexes do not match source table tab")
26342636
replicationtestutils.WaitForAllProducerJobsToFail(t, dbB)
26352637

26362638
// Create the missing indexes on each side and verify the stream can be
@@ -2639,7 +2641,7 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
26392641
dbA.Exec(t, "CREATE UNIQUE INDEX multi_idx ON tab(composite_col, pk)")
26402642
dbB.Exec(t, "CREATE UNIQUE INDEX payload_idx ON b.tab(payload)")
26412643
dbA.QueryRow(t,
2642-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab",
2644+
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab ON $1 INTO TABLE tab WITH MODE = 'validated'",
26432645
dbBURL.String(),
26442646
).Scan(&jobAID)
26452647

@@ -2654,7 +2656,7 @@ func TestLogicalReplicationCreationChecks(t *testing.T) {
26542656
dbB.Exec(t, "CREATE TABLE b.tab2 (pk INT PRIMARY KEY, payload STRING DEFAULT 'dog')")
26552657
dbB.Exec(t, "Insert into tab2 values (1)")
26562658
dbA.QueryRow(t,
2657-
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab2 ON $1 INTO TABLE tab2",
2659+
"CREATE LOGICAL REPLICATION STREAM FROM TABLE tab2 ON $1 INTO TABLE tab2 WITH MODE = 'validated'",
26582660
dbBURL.String(),
26592661
).Scan(&jobAID)
26602662
WaitUntilReplicatedTime(t, s.Clock().Now(), dbA, jobAID)

Diff for: pkg/sql/catalog/tabledesc/logical_replication_helpers.go

+14-3
Original file line numberDiff line numberDiff line change
@@ -23,11 +23,11 @@ import (
2323
// descriptor is a valid target for logical replication and is equivalent to the
2424
// source table.
2525
func CheckLogicalReplicationCompatibility(
26-
src, dst *descpb.TableDescriptor, skipTableEquivalenceCheck bool,
26+
src, dst *descpb.TableDescriptor, skipTableEquivalenceCheck bool, requireKvWriterCompatible bool,
2727
) error {
2828
const cannotLDRMsg = "cannot create logical replication stream"
2929
if !skipTableEquivalenceCheck {
30-
if err := checkSrcDstColsMatch(src, dst); err != nil {
30+
if err := checkSrcDstColsMatch(src, dst, requireKvWriterCompatible); err != nil {
3131
return pgerror.Wrapf(err, pgcode.InvalidTableDefinition, cannotLDRMsg)
3232
}
3333
}
@@ -180,7 +180,12 @@ func checkColumnFamilies(dst *descpb.TableDescriptor) error {
180180
// All column names and types must match with the source table’s columns. The KV
181181
// and SQL write path ingestion side logic assumes that src and dst columns
182182
// match. If they don’t, the LDR job will DLQ these rows and move on.
183-
func checkSrcDstColsMatch(src *descpb.TableDescriptor, dst *descpb.TableDescriptor) error {
183+
//
184+
// If requireKvWriterCompatible is true, we also check that the column IDs
185+
// match.
186+
func checkSrcDstColsMatch(
187+
src *descpb.TableDescriptor, dst *descpb.TableDescriptor, requireKvWriterCompatible bool,
188+
) error {
184189
if len(src.Columns) != len(dst.Columns) {
185190
return errors.Newf(
186191
"destination table %s has %d columns, but the source table %s has %d columns",
@@ -211,6 +216,12 @@ func checkSrcDstColsMatch(src *descpb.TableDescriptor, dst *descpb.TableDescript
211216
dst.Name, dstCol.Name, dstCol.Type.SQLStringForError(), src.Name, srcCol.Type.SQLStringForError(),
212217
)
213218
}
219+
220+
if requireKvWriterCompatible && srcCol.ID != dstCol.ID {
221+
return errors.Newf("destination table %s column %s has ID %d, but the source table %s has ID %d",
222+
dst.Name, dstCol.Name, dstCol.ID, src.Name, srcCol.ID,
223+
)
224+
}
214225
}
215226
return nil
216227
}

0 commit comments

Comments
 (0)