-
Notifications
You must be signed in to change notification settings - Fork 3.8k
/
backfill.go
1240 lines (1128 loc) · 35.9 KB
/
backfill.go
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
486
487
488
489
490
491
492
493
494
495
496
497
498
499
500
501
502
503
504
505
506
507
508
509
510
511
512
513
514
515
516
517
518
519
520
521
522
523
524
525
526
527
528
529
530
531
532
533
534
535
536
537
538
539
540
541
542
543
544
545
546
547
548
549
550
551
552
553
554
555
556
557
558
559
560
561
562
563
564
565
566
567
568
569
570
571
572
573
574
575
576
577
578
579
580
581
582
583
584
585
586
587
588
589
590
591
592
593
594
595
596
597
598
599
600
601
602
603
604
605
606
607
608
609
610
611
612
613
614
615
616
617
618
619
620
621
622
623
624
625
626
627
628
629
630
631
632
633
634
635
636
637
638
639
640
641
642
643
644
645
646
647
648
649
650
651
652
653
654
655
656
657
658
659
660
661
662
663
664
665
666
667
668
669
670
671
672
673
674
675
676
677
678
679
680
681
682
683
684
685
686
687
688
689
690
691
692
693
694
695
696
697
698
699
700
701
702
703
704
705
706
707
708
709
710
711
712
713
714
715
716
717
718
719
720
721
722
723
724
725
726
727
728
729
730
731
732
733
734
735
736
737
738
739
740
741
742
743
744
745
746
747
748
749
750
751
752
753
754
755
756
757
758
759
760
761
762
763
764
765
766
767
768
769
770
771
772
773
774
775
776
777
778
779
780
781
782
783
784
785
786
787
788
789
790
791
792
793
794
795
796
797
798
799
800
801
802
803
804
805
806
807
808
809
810
811
812
813
814
815
816
817
818
819
820
821
822
823
824
825
826
827
828
829
830
831
832
833
834
835
836
837
838
839
840
841
842
843
844
845
846
847
848
849
850
851
852
853
854
855
856
857
858
859
860
861
862
863
864
865
866
867
868
869
870
871
872
873
874
875
876
877
878
879
880
881
882
883
884
885
886
887
888
889
890
891
892
893
894
895
896
897
898
899
900
901
902
903
904
905
906
907
908
909
910
911
912
913
914
915
916
917
918
919
920
921
922
923
924
925
926
927
928
929
930
931
932
933
934
935
936
937
938
939
940
941
942
943
944
945
946
947
948
949
950
951
952
953
954
955
956
957
958
959
960
961
962
963
964
965
966
967
968
969
970
971
972
973
974
975
976
977
978
979
980
981
982
983
984
985
986
987
988
989
990
991
992
993
994
995
996
997
998
999
1000
// Copyright 2015 The Cockroach Authors.
//
// 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,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or
// implied. See the License for the specific language governing
// permissions and limitations under the License.
package sql
import (
"context"
"fmt"
"sort"
"time"
"github.com/cockroachdb/cockroach/pkg/internal/client"
"github.com/cockroachdb/cockroach/pkg/jobs"
"github.com/cockroachdb/cockroach/pkg/kv"
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/settings"
"github.com/cockroachdb/cockroach/pkg/sql/backfill"
"github.com/cockroachdb/cockroach/pkg/sql/distsqlrun"
"github.com/cockroachdb/cockroach/pkg/sql/pgwire/pgerror"
"github.com/cockroachdb/cockroach/pkg/sql/row"
"github.com/cockroachdb/cockroach/pkg/sql/sem/tree"
"github.com/cockroachdb/cockroach/pkg/sql/sqlbase"
"github.com/cockroachdb/cockroach/pkg/util/ctxgroup"
"github.com/cockroachdb/cockroach/pkg/util/hlc"
"github.com/cockroachdb/cockroach/pkg/util/log"
"github.com/cockroachdb/cockroach/pkg/util/timeutil"
"github.com/pkg/errors"
)
const (
// TODO(vivek): Replace these constants with a runtime budget for the
// operation chunk involved.
// columnTruncateAndBackfillChunkSize is the maximum number of columns
// processed per chunk during column truncate or backfill.
columnTruncateAndBackfillChunkSize = 200
// indexTruncateChunkSize is the maximum number of index entries truncated
// per chunk during an index truncation. This value is larger than the
// other chunk constants because the operation involves only running a
// DeleteRange().
indexTruncateChunkSize = 600
// indexTxnBackfillChunkSize is the maximum number index entries backfilled
// per chunk during an index backfill done in a txn. The index backfill
// involves a table scan, and a number of individual ops presented in a batch.
// This value is smaller than ColumnTruncateAndBackfillChunkSize, because it
// involves a number of individual index row updates that can be scattered
// over many ranges.
indexTxnBackfillChunkSize = 100
// checkpointInterval is the interval after which a checkpoint of the
// schema change is posted.
checkpointInterval = 1 * time.Minute
)
var indexBulkBackfillChunkSize = settings.RegisterIntSetting(
"schemachanger.bulk_index_backfill.batch_size",
"number of rows to process at a time during bulk index backfill",
5000000,
)
var _ sort.Interface = columnsByID{}
var _ sort.Interface = indexesByID{}
type columnsByID []sqlbase.ColumnDescriptor
func (cds columnsByID) Len() int {
return len(cds)
}
func (cds columnsByID) Less(i, j int) bool {
return cds[i].ID < cds[j].ID
}
func (cds columnsByID) Swap(i, j int) {
cds[i], cds[j] = cds[j], cds[i]
}
type indexesByID []sqlbase.IndexDescriptor
func (ids indexesByID) Len() int {
return len(ids)
}
func (ids indexesByID) Less(i, j int) bool {
return ids[i].ID < ids[j].ID
}
func (ids indexesByID) Swap(i, j int) {
ids[i], ids[j] = ids[j], ids[i]
}
func (sc *SchemaChanger) getChunkSize(chunkSize int64) int64 {
if sc.testingKnobs.BackfillChunkSize > 0 {
return sc.testingKnobs.BackfillChunkSize
}
return chunkSize
}
// runBackfill runs the backfill for the schema changer.
func (sc *SchemaChanger) runBackfill(
ctx context.Context,
lease *sqlbase.TableDescriptor_SchemaChangeLease,
evalCtx *extendedEvalContext,
) error {
if sc.testingKnobs.RunBeforeBackfill != nil {
if err := sc.testingKnobs.RunBeforeBackfill(); err != nil {
return err
}
}
if err := sc.ExtendLease(ctx, lease); err != nil {
return err
}
// Mutations are applied in a FIFO order. Only apply the first set of
// mutations. Collect the elements that are part of the mutation.
var droppedIndexDescs []sqlbase.IndexDescriptor
var addedIndexDescs []sqlbase.IndexDescriptor
var addedChecks []*sqlbase.TableDescriptor_CheckConstraint
var droppedChecks []*sqlbase.TableDescriptor_CheckConstraint
var checksToValidate []sqlbase.ConstraintToUpdate
var tableDesc *sqlbase.TableDescriptor
if err := sc.db.Txn(ctx, func(ctx context.Context, txn *client.Txn) error {
var err error
tableDesc, err = sqlbase.GetTableDescFromID(ctx, txn, sc.tableID)
return err
}); err != nil {
return err
}
// Short circuit the backfill if the table has been deleted.
if tableDesc.Dropped() {
return nil
}
version := tableDesc.Version
log.Infof(ctx, "Running backfill for %q, v=%d, m=%d",
tableDesc.Name, tableDesc.Version, sc.mutationID)
needColumnBackfill := false
for _, m := range tableDesc.Mutations {
if m.MutationID != sc.mutationID {
break
}
switch m.Direction {
case sqlbase.DescriptorMutation_ADD:
switch t := m.Descriptor_.(type) {
case *sqlbase.DescriptorMutation_Column:
if sqlbase.ColumnNeedsBackfill(m.GetColumn()) {
needColumnBackfill = true
}
case *sqlbase.DescriptorMutation_Index:
addedIndexDescs = append(addedIndexDescs, *t.Index)
case *sqlbase.DescriptorMutation_Constraint:
switch t.Constraint.ConstraintType {
case sqlbase.ConstraintToUpdate_CHECK:
addedChecks = append(addedChecks, &t.Constraint.Check)
checksToValidate = append(checksToValidate, *t.Constraint)
default:
return errors.Errorf("unsupported constraint type: %d", t.Constraint.ConstraintType)
}
default:
return errors.Errorf("unsupported mutation: %+v", m)
}
case sqlbase.DescriptorMutation_DROP:
switch t := m.Descriptor_.(type) {
case *sqlbase.DescriptorMutation_Column:
needColumnBackfill = true
case *sqlbase.DescriptorMutation_Index:
if !sc.canClearRangeForDrop(t.Index) {
droppedIndexDescs = append(droppedIndexDescs, *t.Index)
}
case *sqlbase.DescriptorMutation_Constraint:
// Only possible during a rollback
if !m.Rollback {
panic("trying to drop constraint through schema changer outside of a rollback")
}
switch t.Constraint.ConstraintType {
case sqlbase.ConstraintToUpdate_CHECK:
droppedChecks = append(droppedChecks, &t.Constraint.Check)
default:
return errors.Errorf("unsupported constraint type: %d", t.Constraint.ConstraintType)
}
default:
return errors.Errorf("unsupported mutation: %+v", m)
}
}
}
// First drop constraints and indexes, then add/drop columns, and only then add indexes and constraints.
// Drop check constraints (if this is a rollback).
if len(droppedChecks) > 0 {
desc, err := sc.dropChecksInRollback(ctx, droppedChecks)
if err != nil {
return err
}
version = desc.Version
}
// Drop indexes not to be removed by `ClearRange`.
if len(droppedIndexDescs) > 0 {
if err := sc.truncateIndexes(ctx, lease, version, droppedIndexDescs); err != nil {
return err
}
}
// Add and drop columns.
if needColumnBackfill {
if err := sc.truncateAndBackfillColumns(ctx, evalCtx, lease, version); err != nil {
return err
}
}
// Add new indexes.
if len(addedIndexDescs) > 0 {
// Check if bulk-adding is enabled and supported by indexes (ie non-unique).
if err := sc.backfillIndexes(ctx, evalCtx, lease, version); err != nil {
return err
}
}
// Add check constraints.
if len(addedChecks) > 0 {
if _, err := sc.addChecks(ctx, addedChecks); err != nil {
return err
}
}
// Validate check constraints.
if len(checksToValidate) > 0 {
if err := sc.validateChecks(ctx, evalCtx, lease, checksToValidate); err != nil {
return err
}
}
return nil
}
func (sc *SchemaChanger) addChecks(
ctx context.Context, addedChecks []*sqlbase.TableDescriptor_CheckConstraint,
) (*ImmutableTableDescriptor, error) {
desc, err := sc.leaseMgr.Publish(ctx, sc.tableID,
func(desc *sqlbase.MutableTableDescriptor) error {
for i, added := range addedChecks {
found := false
for _, c := range desc.Checks {
if c.Name == added.Name {
log.VEventf(
ctx, 2,
"backfiller tried to add constraint %+v but found existing constraint %+v, presumably due to a retry",
added, c,
)
found = true
break
}
}
if !found {
desc.Checks = append(desc.Checks, addedChecks[i])
}
}
return nil
},
func(txn *client.Txn) error {
return nil
},
)
if err != nil {
return nil, err
}
if err := sc.waitToUpdateLeases(ctx, sc.tableID); err != nil {
return nil, err
}
return desc, nil
}
func (sc *SchemaChanger) dropChecksInRollback(
ctx context.Context, droppedChecks []*sqlbase.TableDescriptor_CheckConstraint,
) (*ImmutableTableDescriptor, error) {
desc, err := sc.leaseMgr.Publish(ctx, sc.tableID,
func(desc *sqlbase.MutableTableDescriptor) error {
remainingDroppedChecks := make(map[string]struct{})
for _, dropped := range droppedChecks {
remainingDroppedChecks[dropped.Name] = struct{}{}
}
checks := make([]*sqlbase.TableDescriptor_CheckConstraint, 0)
for _, c := range desc.Checks {
found := false
for _, dropped := range droppedChecks {
if dropped.Name == c.Name {
found = true
delete(remainingDroppedChecks, dropped.Name)
break
}
}
if !found {
checks = append(checks, c)
}
}
for droppedName := range remainingDroppedChecks {
log.VEventf(
ctx, 2,
"backfiller tried to drop constraint %+v which does not exist, presumably due to a retry",
droppedName,
)
}
desc.Checks = checks
return nil
},
func(txn *client.Txn) error {
return nil
},
)
if err != nil {
return nil, err
}
if err := sc.waitToUpdateLeases(ctx, sc.tableID); err != nil {
return nil, err
}
return desc, nil
}
func (sc *SchemaChanger) validateChecks(
ctx context.Context,
evalCtx *extendedEvalContext,
lease *sqlbase.TableDescriptor_SchemaChangeLease,
checks []sqlbase.ConstraintToUpdate,
) error {
if testDisableTableLeases {
return nil
}
readAsOf := sc.clock.Now()
return sc.db.Txn(ctx, func(ctx context.Context, txn *client.Txn) error {
txn.SetFixedTimestamp(ctx, readAsOf)
tableDesc, err := sqlbase.GetTableDescFromID(ctx, txn, sc.tableID)
if err != nil {
return err
}
if err := sc.ExtendLease(ctx, lease); err != nil {
return err
}
grp := ctxgroup.WithContext(ctx)
// Notify when validation is finished (or has returned an error) for a check.
countDone := make(chan struct{}, len(checks))
for _, c := range checks {
grp.GoCtx(func(ctx context.Context) error {
defer func() { countDone <- struct{}{} }()
// Make the mutations public in a private copy of the descriptor
// and add it to the TableCollection, so that we can use SQL below to perform
// the validation. We wouldn't have needed to do this if we could have
// updated the descriptor and run validation in the same transaction. However,
// our current system is incapable of running long running schema changes
// (the validation can take many minutes). So we pretend that the schema
// has been updated and actually update it in a separate transaction that
// follows this one.
desc, err := sqlbase.NewImmutableTableDescriptor(*tableDesc).MakeFirstMutationPublic()
if err != nil {
return err
}
// Create a new eval context only because the eval context cannot be shared across many
// goroutines.
newEvalCtx := createSchemaChangeEvalCtx(ctx, readAsOf, evalCtx.Tracing, sc.ieFactory)
return validateCheckInTxn(ctx, sc.leaseMgr, &newEvalCtx.EvalContext, desc, txn, &c.Name)
})
}
// Periodic schema change lease extension.
grp.GoCtx(func(ctx context.Context) error {
count := len(checks)
refreshTimer := timeutil.NewTimer()
defer refreshTimer.Stop()
refreshTimer.Reset(checkpointInterval)
for {
select {
case <-countDone:
count--
if count == 0 {
// Stop.
return nil
}
case <-refreshTimer.C:
refreshTimer.Read = true
refreshTimer.Reset(checkpointInterval)
if err := sc.ExtendLease(ctx, lease); err != nil {
return err
}
case <-ctx.Done():
return ctx.Err()
}
}
})
return grp.Wait()
})
}
func (sc *SchemaChanger) getTableVersion(
ctx context.Context, txn *client.Txn, tc *TableCollection, version sqlbase.DescriptorVersion,
) (*sqlbase.ImmutableTableDescriptor, error) {
tableDesc, err := tc.getTableVersionByID(ctx, txn, sc.tableID, ObjectLookupFlags{})
if err != nil {
return nil, err
}
if version != tableDesc.Version {
return nil, makeErrTableVersionMismatch(tableDesc.Version, version)
}
return tableDesc, nil
}
func (sc *SchemaChanger) truncateIndexes(
ctx context.Context,
lease *sqlbase.TableDescriptor_SchemaChangeLease,
version sqlbase.DescriptorVersion,
dropped []sqlbase.IndexDescriptor,
) error {
chunkSize := sc.getChunkSize(indexTruncateChunkSize)
if sc.testingKnobs.BackfillChunkSize > 0 {
chunkSize = sc.testingKnobs.BackfillChunkSize
}
alloc := &sqlbase.DatumAlloc{}
for _, desc := range dropped {
var resume roachpb.Span
for rowIdx, done := int64(0), false; !done; rowIdx += chunkSize {
// First extend the schema change lease.
if err := sc.ExtendLease(ctx, lease); err != nil {
return err
}
resumeAt := resume
if log.V(2) {
log.Infof(ctx, "drop index (%d, %d) at row: %d, span: %s",
sc.tableID, sc.mutationID, rowIdx, resume)
}
if err := sc.db.Txn(ctx, func(ctx context.Context, txn *client.Txn) error {
if fn := sc.execCfg.DistSQLRunTestingKnobs.RunBeforeBackfillChunk; fn != nil {
if err := fn(resume); err != nil {
return err
}
}
if fn := sc.execCfg.DistSQLRunTestingKnobs.RunAfterBackfillChunk; fn != nil {
defer fn()
}
tc := &TableCollection{leaseMgr: sc.leaseMgr}
defer tc.releaseTables(ctx)
tableDesc, err := sc.getTableVersion(ctx, txn, tc, version)
if err != nil {
return err
}
rd, err := row.MakeDeleter(
txn, tableDesc, nil, nil, row.SkipFKs, nil /* *tree.EvalContext */, alloc,
)
if err != nil {
return err
}
td := tableDeleter{rd: rd, alloc: alloc}
if err := td.init(txn, nil /* *tree.EvalContext */); err != nil {
return err
}
if !sc.canClearRangeForDrop(&desc) {
resume, err = td.deleteIndex(
ctx,
&desc,
resumeAt,
chunkSize,
false, /* traceKV */
)
done = resume.Key == nil
return err
}
done = true
return td.clearIndex(ctx, &desc)
}); err != nil {
return err
}
}
if err := sc.db.Txn(ctx, func(ctx context.Context, txn *client.Txn) error {
return removeIndexZoneConfigs(ctx, txn, sc.execCfg, sc.tableID, dropped)
}); err != nil {
return err
}
}
return nil
}
type backfillType int
const (
_ backfillType = iota
columnBackfill
indexBackfill
)
// getJobIDForMutationWithDescriptor returns a job id associated with a mutation given
// a table descriptor. Unlike getJobIDForMutation this doesn't need transaction.
func getJobIDForMutationWithDescriptor(
ctx context.Context, tableDesc *sqlbase.TableDescriptor, mutationID sqlbase.MutationID,
) (int64, error) {
for _, job := range tableDesc.MutationJobs {
if job.MutationID == mutationID {
return job.JobID, nil
}
}
return 0, errors.Errorf("job not found for table id %d, mutation %d", tableDesc.ID, mutationID)
}
// nRanges returns the number of ranges that cover a set of spans.
func (sc *SchemaChanger) nRanges(
ctx context.Context, txn *client.Txn, spans []roachpb.Span,
) (int, error) {
spanResolver := sc.distSQLPlanner.spanResolver.NewSpanResolverIterator(txn)
rangeIds := make(map[int64]struct{})
for _, span := range spans {
// For each span, iterate the spanResolver until it's exhausted, storing
// the found range ids in the map to de-duplicate them.
spanResolver.Seek(ctx, span, kv.Ascending)
for {
if !spanResolver.Valid() {
return 0, spanResolver.Error()
}
rangeIds[int64(spanResolver.Desc().RangeID)] = struct{}{}
if !spanResolver.NeedAnother() {
break
}
spanResolver.Next(ctx)
}
}
return len(rangeIds), nil
}
// distBackfill runs (or continues) a backfill for the first mutation
// enqueued on the SchemaChanger's table descriptor that passes the input
// MutationFilter.
func (sc *SchemaChanger) distBackfill(
ctx context.Context,
evalCtx *extendedEvalContext,
lease *sqlbase.TableDescriptor_SchemaChangeLease,
version sqlbase.DescriptorVersion,
backfillType backfillType,
backfillChunkSize int64,
filter backfill.MutationFilter,
) error {
duration := checkpointInterval
if sc.testingKnobs.WriteCheckpointInterval > 0 {
duration = sc.testingKnobs.WriteCheckpointInterval
}
chunkSize := sc.getChunkSize(backfillChunkSize)
origNRanges := -1
origFractionCompleted := sc.job.FractionCompleted()
fractionLeft := 1 - origFractionCompleted
readAsOf := sc.clock.Now()
for {
var spans []roachpb.Span
if err := sc.db.Txn(ctx, func(ctx context.Context, txn *client.Txn) error {
var err error
spans, _, _, err = distsqlrun.GetResumeSpans(
ctx, sc.jobRegistry, txn, sc.tableID, sc.mutationID, filter)
return err
}); err != nil {
return err
}
if len(spans) <= 0 {
break
}
if err := sc.ExtendLease(ctx, lease); err != nil {
return err
}
log.VEventf(ctx, 2, "backfill: process %+v spans", spans)
if err := sc.db.Txn(ctx, func(ctx context.Context, txn *client.Txn) error {
// Report schema change progress. We define progress at this point
// as the the fraction of fully-backfilled ranges of the primary index of
// the table being scanned. Since we may have already modified the
// fraction completed of our job from the 10% allocated to completing the
// schema change state machine or from a previous backfill attempt,
// we scale that fraction of ranges completed by the remaining fraction
// of the job's progress bar.
nRanges, err := sc.nRanges(ctx, txn, spans)
if err != nil {
return err
}
if origNRanges == -1 {
origNRanges = nRanges
}
if nRanges < origNRanges {
fractionRangesFinished := float32(origNRanges-nRanges) / float32(origNRanges)
fractionCompleted := origFractionCompleted + fractionLeft*fractionRangesFinished
if err := sc.job.FractionProgressed(ctx, jobs.FractionUpdater(fractionCompleted)); err != nil {
return jobs.SimplifyInvalidStatusError(err)
}
}
tc := &TableCollection{leaseMgr: sc.leaseMgr}
// Use a leased table descriptor for the backfill.
defer tc.releaseTables(ctx)
tableDesc, err := sc.getTableVersion(ctx, txn, tc, version)
if err != nil {
return err
}
// otherTableDescs contains any other table descriptors required by the
// backfiller processor.
var otherTableDescs []sqlbase.TableDescriptor
if backfillType == columnBackfill {
fkTables, err := row.MakeFkMetadata(
ctx,
tableDesc,
row.CheckUpdates,
row.NoLookup,
row.NoCheckPrivilege,
nil, /* AnalyzeExprFunction */
nil, /* CheckHelper */
)
if err != nil {
return err
}
for k := range fkTables {
table, err := tc.getTableVersionByID(ctx, txn, k, ObjectLookupFlags{})
if err != nil {
return err
}
otherTableDescs = append(otherTableDescs, *table.TableDesc())
}
}
rw := &errOnlyResultWriter{}
recv := MakeDistSQLReceiver(
ctx,
rw,
tree.Rows, /* stmtType - doesn't matter here since no result are produced */
sc.rangeDescriptorCache,
sc.leaseHolderCache,
nil, /* txn - the flow does not run wholly in a txn */
func(ts hlc.Timestamp) {
_ = sc.clock.Update(ts)
},
evalCtx.Tracing,
)
defer recv.Release()
planCtx := sc.distSQLPlanner.NewPlanningCtx(ctx, evalCtx, txn)
plan, err := sc.distSQLPlanner.createBackfiller(
planCtx, backfillType, *tableDesc.TableDesc(), duration, chunkSize, spans, otherTableDescs, readAsOf,
)
if err != nil {
return err
}
sc.distSQLPlanner.Run(
planCtx,
nil, /* txn - the processors manage their own transactions */
&plan, recv, evalCtx,
nil, /* finishedSetupFn */
)
return rw.Err()
}); err != nil {
return err
}
}
return nil
}
// validate the new indexes being added
func (sc *SchemaChanger) validateIndexes(
ctx context.Context,
evalCtx *extendedEvalContext,
lease *sqlbase.TableDescriptor_SchemaChangeLease,
) error {
if testDisableTableLeases {
return nil
}
readAsOf := sc.clock.Now()
return sc.db.Txn(ctx, func(ctx context.Context, txn *client.Txn) error {
txn.SetFixedTimestamp(ctx, readAsOf)
tableDesc, err := sqlbase.GetTableDescFromID(ctx, txn, sc.tableID)
if err != nil {
return err
}
if err := sc.ExtendLease(ctx, lease); err != nil {
return err
}
var forwardIndexes []*sqlbase.IndexDescriptor
var invertedIndexes []*sqlbase.IndexDescriptor
for _, m := range tableDesc.Mutations {
if sc.mutationID != m.MutationID {
break
}
idx := m.GetIndex()
if idx == nil || m.Direction == sqlbase.DescriptorMutation_DROP {
continue
}
switch idx.Type {
case sqlbase.IndexDescriptor_FORWARD:
forwardIndexes = append(forwardIndexes, idx)
case sqlbase.IndexDescriptor_INVERTED:
invertedIndexes = append(invertedIndexes, idx)
}
}
if len(forwardIndexes) == 0 && len(invertedIndexes) == 0 {
return nil
}
grp := ctxgroup.WithContext(ctx)
forwardIndexesDone := make(chan struct{})
invertedIndexesDone := make(chan struct{})
grp.GoCtx(func(ctx context.Context) error {
defer close(forwardIndexesDone)
if len(forwardIndexes) > 0 {
return sc.validateForwardIndexes(ctx, evalCtx, txn, tableDesc, readAsOf, forwardIndexes)
}
return nil
})
grp.GoCtx(func(ctx context.Context) error {
defer close(invertedIndexesDone)
if len(invertedIndexes) > 0 {
return sc.validateInvertedIndexes(ctx, evalCtx, txn, tableDesc, readAsOf, invertedIndexes)
}
return nil
})
// Periodic schema change lease extension.
grp.GoCtx(func(ctx context.Context) error {
forwardDone := false
invertedDone := false
refreshTimer := timeutil.NewTimer()
defer refreshTimer.Stop()
refreshTimer.Reset(checkpointInterval)
for {
if forwardDone && invertedDone {
return nil
}
select {
case <-forwardIndexesDone:
forwardDone = true
case <-invertedIndexesDone:
invertedDone = true
case <-refreshTimer.C:
refreshTimer.Read = true
refreshTimer.Reset(checkpointInterval)
if err := sc.ExtendLease(ctx, lease); err != nil {
return err
}
case <-ctx.Done():
return ctx.Err()
}
}
})
return grp.Wait()
})
}
func (sc *SchemaChanger) validateInvertedIndexes(
ctx context.Context,
evalCtx *extendedEvalContext,
txn *client.Txn,
tableDesc *TableDescriptor,
readAsOf hlc.Timestamp,
indexes []*sqlbase.IndexDescriptor,
) error {
grp := ctxgroup.WithContext(ctx)
expectedCount := make([]int64, len(indexes))
countReady := make([]chan struct{}, len(indexes))
for i, idx := range indexes {
i, idx := i, idx
countReady[i] = make(chan struct{})
grp.GoCtx(func(ctx context.Context) error {
// Inverted indexes currently can't be interleaved, so a KV scan can be
// used to get the index length.
// TODO (lucy): Switch to using DistSQL to get the count, so that we get
// distributed execution and avoid bypassing the SQL decoding
start := timeutil.Now()
var idxLen int64
key := tableDesc.IndexSpan(idx.ID).Key
endKey := tableDesc.IndexSpan(idx.ID).EndKey
for {
kvs, err := txn.Scan(ctx, key, endKey, 1000000)
if err != nil {
return err
}
if len(kvs) == 0 {
break
}
idxLen += int64(len(kvs))
key = kvs[len(kvs)-1].Key.PrefixEnd()
}
log.Infof(ctx, "inverted index %s/%s count = %d, took %s",
tableDesc.Name, idx.Name, idxLen, timeutil.Since(start))
select {
case <-countReady[i]:
if idxLen != expectedCount[i] {
// JSON columns cannot have unique indexes, so if the expected and
// actual counts do not match, it's always a bug rather than a
// uniqueness violation.
return errors.Errorf("validation of index %s failed: expected %d rows, found %d",
idx.Name, expectedCount[i], idxLen)
}
case <-ctx.Done():
return ctx.Err()
}
return nil
})
grp.GoCtx(func(ctx context.Context) error {
defer close(countReady[i])
start := timeutil.Now()
if len(idx.ColumnNames) != 1 {
panic(fmt.Sprintf("expected inverted index %s to have exactly 1 column, but found columns %+v",
idx.Name, idx.ColumnNames))
}
col := idx.ColumnNames[0]
row, err := evalCtx.InternalExecutor.QueryRow(ctx, "verify-inverted-idx-count", txn,
fmt.Sprintf(
`SELECT coalesce(sum_int(crdb_internal.json_num_index_entries(%s)), 0) FROM [%d AS t]`,
col, tableDesc.ID,
),
)
if err != nil {
return err
}
expectedCount[i] = int64(tree.MustBeDInt(row[0]))
log.Infof(ctx, "JSON column %s/%s expected inverted index count = %d, took %s",
tableDesc.Name, col, expectedCount[i], timeutil.Since(start))
return nil
})
}
return grp.Wait()
}
func (sc *SchemaChanger) validateForwardIndexes(
ctx context.Context,
evalCtx *extendedEvalContext,
txn *client.Txn,
tableDesc *TableDescriptor,
readAsOf hlc.Timestamp,
indexes []*sqlbase.IndexDescriptor,
) error {
grp := ctxgroup.WithContext(ctx)
var tableRowCount int64
// Close when table count is ready.
tableCountReady := make(chan struct{})
// Compute the size of each index.
for _, idx := range indexes {
idx := idx
grp.GoCtx(func(ctx context.Context) error {
start := timeutil.Now()
// Make the mutations public in a private copy of the descriptor
// and add it to the TableCollection, so that we can use SQL below to perform
// the validation. We wouldn't have needed to do this if we could have
// updated the descriptor and run validation in the same transaction. However,
// our current system is incapable of running long running schema changes
// (the validation can take many minutes). So we pretend that the schema
// has been updated and actually update it in a separate transaction that
// follows this one.
desc, err := sqlbase.NewImmutableTableDescriptor(*tableDesc).MakeFirstMutationPublic()
if err != nil {
return err
}
tc := &TableCollection{leaseMgr: sc.leaseMgr}
// pretend that the schema has been modified.
if err := tc.addUncommittedTable(*desc); err != nil {
return err
}
// Create a new eval context only because the eval context cannot be shared across many
// goroutines.
newEvalCtx := createSchemaChangeEvalCtx(ctx, readAsOf, evalCtx.Tracing, sc.ieFactory)
// TODO(vivek): This is not a great API. Leaving #34304 open.
ie := newEvalCtx.InternalExecutor.(*SessionBoundInternalExecutor)
ie.impl.tcModifier = tc
defer func() {
ie.impl.tcModifier = nil
}()
row, err := newEvalCtx.InternalExecutor.QueryRow(ctx, "verify-idx-count", txn,
fmt.Sprintf(`SELECT count(*) FROM [%d AS t]@[%d]`, tableDesc.ID, idx.ID))
if err != nil {
return err
}
idxLen := int64(tree.MustBeDInt(row[0]))
log.Infof(ctx, "index %s/%s row count = %d, took %s",
tableDesc.Name, idx.Name, idxLen, timeutil.Since(start))
select {
case <-tableCountReady:
if idxLen != tableRowCount {
// TODO(vivek): find the offending row and include it in the error.
return pgerror.NewErrorf(
pgerror.CodeUniqueViolationError,
"%d entries, expected %d violates unique constraint %q",
idxLen, tableRowCount, idx.Name,
)
}
case <-ctx.Done():
return ctx.Err()
}
return nil
})
}
grp.GoCtx(func(ctx context.Context) error {
defer close(tableCountReady)
var tableRowCountTime time.Duration
start := timeutil.Now()
// Count the number of rows in the table.
cnt, err := evalCtx.InternalExecutor.QueryRow(ctx, "VERIFY INDEX", txn,
fmt.Sprintf(`SELECT count(1) FROM [%d AS t]`, tableDesc.ID))
if err != nil {
return err
}
tableRowCount = int64(tree.MustBeDInt(cnt[0]))
tableRowCountTime = timeutil.Since(start)
log.Infof(ctx, "table %s row count = %d, took %s",
tableDesc.Name, tableRowCount, tableRowCountTime)
return nil
})
return grp.Wait()
}
func (sc *SchemaChanger) backfillIndexes(
ctx context.Context,
evalCtx *extendedEvalContext,
lease *sqlbase.TableDescriptor_SchemaChangeLease,
version sqlbase.DescriptorVersion,
) error {
if fn := sc.testingKnobs.RunBeforeIndexBackfill; fn != nil {
fn()
}
chunkSize := int64(indexTxnBackfillChunkSize)
bulk := backfill.BulkWriteIndex.Get(&sc.settings.SV)
if bulk {
chunkSize = indexBulkBackfillChunkSize.Get(&sc.settings.SV)
}
if err := sc.distBackfill(
ctx, evalCtx, lease, version, indexBackfill, chunkSize,
backfill.IndexMutationFilter); err != nil {
return err
}
return sc.validateIndexes(ctx, evalCtx, lease)
}
func (sc *SchemaChanger) truncateAndBackfillColumns(
ctx context.Context,
evalCtx *extendedEvalContext,
lease *sqlbase.TableDescriptor_SchemaChangeLease,
version sqlbase.DescriptorVersion,
) error {
return sc.distBackfill(
ctx, evalCtx,
lease, version, columnBackfill, columnTruncateAndBackfillChunkSize,
backfill.ColumnMutationFilter)
}
// runSchemaChangesInTxn runs all the schema changes immediately in a
// transaction. This is called when a CREATE TABLE is followed by
// schema changes in the same transaction. The CREATE TABLE is
// invisible to the rest of the cluster, so the schema changes
// can be executed immediately on the same version of the table.
func runSchemaChangesInTxn(
ctx context.Context,