-
Notifications
You must be signed in to change notification settings - Fork 289
/
Copy pathloader.go
1544 lines (1351 loc) · 42.7 KB
/
loader.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 2019 PingCAP, Inc.
//
// 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,
// See the License for the specific language governing permissions and
// limitations under the License.
package loader
import (
"bufio"
"bytes"
"context"
"encoding/hex"
"io"
"os"
"path/filepath"
"strings"
"sync"
"time"
"github.com/pingcap/errors"
"github.com/pingcap/failpoint"
cm "github.com/pingcap/tidb-tools/pkg/column-mapping"
"github.com/pingcap/tidb/dumpling/export"
"github.com/pingcap/tidb/util/filter"
regexprrouter "github.com/pingcap/tidb/util/regexpr-router"
router "github.com/pingcap/tidb/util/table-router"
"github.com/pingcap/tiflow/dm/config"
"github.com/pingcap/tiflow/dm/config/dbconfig"
"github.com/pingcap/tiflow/dm/pb"
"github.com/pingcap/tiflow/dm/pkg/conn"
tcontext "github.com/pingcap/tiflow/dm/pkg/context"
fr "github.com/pingcap/tiflow/dm/pkg/func-rollback"
"github.com/pingcap/tiflow/dm/pkg/log"
"github.com/pingcap/tiflow/dm/pkg/terror"
"github.com/pingcap/tiflow/dm/pkg/utils"
"github.com/pingcap/tiflow/dm/unit"
clientv3 "go.etcd.io/etcd/client/v3"
"go.uber.org/atomic"
"go.uber.org/zap"
"golang.org/x/sync/errgroup"
)
const (
jobCount = 1000
uninitializedOffset = -1
)
// FilePosSet represents a set in mathematics.
type FilePosSet map[string][]int64
// DataFiles represent all data files for a single table.
type DataFiles []string
// Tables2DataFiles represent all data files of a table collection as a map.
type Tables2DataFiles map[string]DataFiles
type dataJob struct {
sql string
schema string
table string
sourceTable string
sourceSchema string
file string
absPath string
offset int64
lastOffset int64
}
type fileJob struct {
schema string
table string
dataFile string
offset int64
info *tableInfo
}
// Worker represents a worker.
type Worker struct {
id int
cfg *config.SubTaskConfig
checkPoint CheckPoint
conn *DBConn
wg sync.WaitGroup
jobQueue chan *dataJob
loader *Loader
logger log.Logger
closed atomic.Bool
}
// NewWorker returns a Worker.
func NewWorker(loader *Loader, id int) *Worker {
w := &Worker{
id: id,
cfg: loader.cfg,
checkPoint: loader.checkPoint,
conn: loader.toDBConns[id],
jobQueue: make(chan *dataJob, jobCount),
loader: loader,
logger: loader.logger.WithFields(zap.Int("worker ID", id)),
}
failpoint.Inject("workerChanSize", func(val failpoint.Value) {
size := val.(int)
w.logger.Info("", zap.String("failpoint", "workerChanSize"), zap.Int("size", size))
w.jobQueue = make(chan *dataJob, size)
})
return w
}
// Close closes worker.
func (w *Worker) Close() {
// simulate the case that doesn't wait all doJob goroutine exit
failpoint.Inject("workerCantClose", func(_ failpoint.Value) {
w.logger.Info("", zap.String("failpoint", "workerCantClose"))
failpoint.Return()
})
if !w.closed.CAS(false, true) {
w.wg.Wait()
w.logger.Info("already closed...")
return
}
w.logger.Info("start to close...")
close(w.jobQueue)
w.wg.Wait()
w.logger.Info("closed !!!")
}
func (w *Worker) run(ctx context.Context, fileJobQueue chan *fileJob, runFatalChan chan *pb.ProcessError) {
w.closed.Store(false)
newCtx, cancel := context.WithCancel(ctx)
defer func() {
cancel()
// make sure all doJob goroutines exit
w.Close()
}()
ctctx := tcontext.NewContext(newCtx, w.logger)
doJob := func() {
hasError := false
for {
job, ok := <-w.jobQueue
if !ok {
w.logger.Info("job queue was closed, execution goroutine exits")
return
}
if job == nil {
w.logger.Info("jobs are finished, execution goroutine exits")
return
}
if hasError {
continue // continue to read so than the sender will not be blocked
}
sqls := make([]string, 0, 3)
sqls = append(sqls, "USE `"+unescapePercent(job.schema, w.logger)+"`;")
sqls = append(sqls, job.sql)
offsetSQL := w.checkPoint.GenSQL(job.file, job.offset)
sqls = append(sqls, offsetSQL)
failpoint.Inject("LoadExceedOffsetExit", func(val failpoint.Value) {
threshold, _ := val.(int)
if job.offset >= int64(threshold) {
w.logger.Warn("load offset execeeds threshold, it will exit", zap.Int64("load offset", job.offset), zap.Int("value", threshold), zap.String("failpoint", "LoadExceedOffsetExit"))
utils.OsExit(1)
}
})
failpoint.Inject("LoadDataSlowDown", nil)
failpoint.Inject("LoadDataSlowDownByTask", func(val failpoint.Value) {
tasks := val.(string)
taskNames := strings.Split(tasks, ",")
for _, taskName := range taskNames {
if w.cfg.Name == taskName {
w.logger.Info("inject failpoint LoadDataSlowDownByTask", zap.String("task", taskName))
<-newCtx.Done()
}
}
})
startTime := time.Now()
err := w.conn.executeSQL(ctctx, sqls)
failpoint.Inject("executeSQLError", func(_ failpoint.Value) {
w.logger.Info("", zap.String("failpoint", "executeSQLError"))
err = errors.New("inject failpoint executeSQLError")
})
if err != nil {
// expect pause rather than exit
err = terror.WithScope(terror.Annotatef(err, "file %s", job.file), terror.ScopeDownstream)
if !utils.IsContextCanceledError(err) {
runFatalChan <- unit.NewProcessError(err)
}
hasError = true
failpoint.Inject("returnDoJobError", func(_ failpoint.Value) {
w.logger.Info("", zap.String("failpoint", "returnDoJobError"))
failpoint.Return()
})
continue
}
txnHistogram.WithLabelValues(w.cfg.Name, w.cfg.WorkerName, w.cfg.SourceID, job.schema, job.table).Observe(time.Since(startTime).Seconds())
failpoint.Inject("loaderCPUpdateOffsetError", func(_ failpoint.Value) {
job.file = "notafile" + job.file
})
if err := w.loader.checkPoint.UpdateOffset(job.file, job.offset); err != nil {
runFatalChan <- unit.NewProcessError(err)
hasError = true
continue
}
// update finished offset after checkpoint updated
w.loader.finishedDataSize.Add(job.offset - job.lastOffset)
if _, ok := w.loader.dbTableDataFinishedSize[job.sourceSchema]; ok {
if _, ok := w.loader.dbTableDataFinishedSize[job.sourceSchema][job.sourceTable]; ok {
w.loader.dbTableDataFinishedSize[job.sourceSchema][job.sourceTable].Store(job.offset)
}
}
}
}
// worker main routine
for {
select {
case <-newCtx.Done():
w.logger.Info("context canceled, main goroutine exits")
return
case job, ok := <-fileJobQueue:
if !ok {
w.logger.Info("file queue was closed, main routine exit.")
return
}
w.wg.Add(1)
go func() {
defer w.wg.Done()
doJob()
}()
// restore a table
if err := w.restoreDataFile(ctx, filepath.Join(w.cfg.Dir, job.dataFile), job.offset, job.info); err != nil {
// expect pause rather than exit
err = terror.Annotatef(err, "restore data file (%v) failed", job.dataFile)
if !utils.IsContextCanceledError(err) {
runFatalChan <- unit.NewProcessError(err)
}
return
}
}
}
}
func (w *Worker) restoreDataFile(ctx context.Context, filePath string, offset int64, table *tableInfo) error {
w.logger.Info("start to restore dump sql file", zap.String("data file", filePath))
err := w.dispatchSQL(ctx, filePath, offset, table)
if err != nil {
return err
}
failpoint.Inject("dispatchError", func(_ failpoint.Value) {
w.logger.Info("", zap.String("failpoint", "dispatchError"))
failpoint.Return(errors.New("inject failpoint dispatchError"))
})
// dispatchSQL completed, send nil to make sure all dmls are applied to target database
// we don't want to close and re-make chan frequently
// but if we need to re-call w.run, we need re-make jobQueue chan
w.jobQueue <- nil
w.wg.Wait()
w.logger.Info("finish to restore dump sql file", zap.String("data file", filePath))
return nil
}
func (w *Worker) dispatchSQL(ctx context.Context, file string, offset int64, table *tableInfo) error {
var (
f *os.File
err error
cur int64
)
baseFile := filepath.Base(file)
f, err = os.Open(file)
if err != nil {
return terror.ErrLoadUnitDispatchSQLFromFile.Delegate(err)
}
defer f.Close()
// file was not found in checkpoint
if offset == uninitializedOffset {
offset = 0
finfo, err2 := f.Stat()
if err2 != nil {
return terror.ErrLoadUnitDispatchSQLFromFile.Delegate(err2)
}
tctx := tcontext.NewContext(ctx, w.logger)
err2 = w.checkPoint.Init(tctx, baseFile, finfo.Size())
failpoint.Inject("WaitLoaderStopAfterInitCheckpoint", func(v failpoint.Value) {
t := v.(int)
w.logger.Info("wait loader stop after init checkpoint")
w.wg.Add(1)
time.Sleep(time.Duration(t) * time.Second)
w.wg.Done()
})
if err2 != nil {
w.logger.Error("fail to initialize checkpoint", zap.String("data file", file), zap.Int64("offset", offset), log.ShortError(err2))
return err2
}
}
cur, err = f.Seek(offset, io.SeekStart)
if err != nil {
return terror.ErrLoadUnitDispatchSQLFromFile.Delegate(err)
}
w.logger.Debug("read file", zap.String("data file", file), zap.Int64("offset", offset))
lastOffset := cur
data := make([]byte, 0, 1024*1024)
br := bufio.NewReader(f)
for {
select {
case <-ctx.Done():
w.logger.Info("sql dispatcher is ready to quit.", zap.String("data file", file), zap.Int64("offset", offset))
return nil
default:
// do nothing
}
line, err := br.ReadString('\n')
if err == io.EOF {
w.logger.Info("data are scanned finished.", zap.String("data file", file), zap.Int64("offset", offset))
break
}
cur += int64(len(line))
realLine := strings.TrimSpace(line[:len(line)-1])
if len(realLine) == 0 {
continue
}
data = append(data, []byte(line)...)
if realLine[len(realLine)-1] == ';' {
query := strings.TrimSpace(string(data))
if strings.HasPrefix(query, "/*") && strings.HasSuffix(query, "*/;") {
data = data[0:0]
continue
}
// extend column also need use reassemble to write SQL and the table name has been renamed
if w.loader.columnMapping != nil || len(table.extendCol) > 0 {
// column mapping and route table
query, err = reassemble(data, table, w.loader.columnMapping)
if err != nil {
return terror.Annotatef(err, "file %s", file)
}
} else if table.sourceTable != table.targetTable {
// dumped data files always use backquote as quotes
query = renameShardingTable(query, table.sourceTable, table.targetTable, false)
}
idx := strings.Index(query, "INSERT INTO")
if idx < 0 {
return terror.ErrLoadUnitInvalidInsertSQL.Generate(query)
}
data = data[0:0]
j := &dataJob{
sql: query,
schema: table.targetSchema,
table: table.targetTable,
sourceSchema: table.sourceSchema,
sourceTable: table.sourceTable,
file: baseFile,
absPath: file,
offset: cur,
lastOffset: lastOffset,
}
lastOffset = cur
w.jobQueue <- j
}
}
return nil
}
type tableInfo struct {
sourceSchema string
sourceTable string
targetSchema string
targetTable string
columnNameList []string
insertHeadStmt string
extendCol []string
extendVal []string
}
// Loader can load your mydumper data into TiDB database.
type Loader struct {
sync.RWMutex
cfg *config.SubTaskConfig
cli *clientv3.Client
workerName string
checkPoint CheckPoint
logger log.Logger
// db -> tables
// table -> data files
db2Tables map[string]Tables2DataFiles
tableInfos map[string]*tableInfo
fileJobQueue chan *fileJob
tableRouter *regexprrouter.RouteTable
baList *filter.Filter
columnMapping *cm.Mapping
toDB *conn.BaseDB
toDBConns []*DBConn
totalFileCount atomic.Int64 // schema + table + data
totalDataSize atomic.Int64
finishedDataSize atomic.Int64
// to calculate remainingTimeGauge metric, map will be init in `l.prepare.prepareDataFiles`
dbTableDataTotalSize map[string]map[string]*atomic.Int64
dbTableDataFinishedSize map[string]map[string]*atomic.Int64
dbTableDataLastFinishedSize map[string]map[string]*atomic.Int64
dbTableDataLastUpdatedTime atomic.Time
speedRecorder *export.SpeedRecorder
metaBinlog atomic.String
metaBinlogGTID atomic.String
// record process error rather than log.Fatal
runFatalChan chan *pb.ProcessError
// for every worker goroutine, not for every data file
workerWg *sync.WaitGroup
// for other goroutines
wg sync.WaitGroup
fileJobQueueClosed atomic.Bool
finish atomic.Bool
closed atomic.Bool
}
// NewLoader creates a new Loader.
func NewLoader(cfg *config.SubTaskConfig, cli *clientv3.Client, workerName string) *Loader {
loader := &Loader{
cfg: cfg,
cli: cli,
db2Tables: make(map[string]Tables2DataFiles),
tableInfos: make(map[string]*tableInfo),
workerWg: new(sync.WaitGroup),
logger: log.With(zap.String("task", cfg.Name), zap.String("unit", "load")),
workerName: workerName,
speedRecorder: export.NewSpeedRecorder(),
}
loader.fileJobQueueClosed.Store(true) // not open yet
return loader
}
// Type implements Unit.Type.
func (l *Loader) Type() pb.UnitType {
return pb.UnitType_Load
}
// Init initializes loader for a load task, but not start Process.
// if fail, it should not call l.Close.
func (l *Loader) Init(ctx context.Context) (err error) {
rollbackHolder := fr.NewRollbackHolder("loader")
defer func() {
if err != nil {
rollbackHolder.RollbackReverseOrder()
}
}()
tctx := tcontext.NewContext(ctx, l.logger)
checkpoint, err := newRemoteCheckPoint(tctx, l.cfg, l.checkpointID())
failpoint.Inject("ignoreLoadCheckpointErr", func(_ failpoint.Value) {
l.logger.Info("", zap.String("failpoint", "ignoreLoadCheckpointErr"))
err = nil
})
if err != nil {
return err
}
l.checkPoint = checkpoint
rollbackHolder.Add(fr.FuncRollback{Name: "close-checkpoint", Fn: l.checkPoint.Close})
l.baList, err = filter.New(l.cfg.CaseSensitive, l.cfg.BAList)
if err != nil {
return terror.ErrLoadUnitGenBAList.Delegate(err)
}
err = l.genRouter(l.cfg.RouteRules)
if err != nil {
return err
}
if len(l.cfg.ColumnMappingRules) > 0 {
l.columnMapping, err = cm.NewMapping(l.cfg.CaseSensitive, l.cfg.ColumnMappingRules)
if err != nil {
return terror.ErrLoadUnitGenColumnMapping.Delegate(err)
}
}
dbCfg := l.cfg.To
dbCfg.RawDBCfg = dbconfig.DefaultRawDBConfig().
SetMaxIdleConns(l.cfg.PoolSize)
// used to change loader's specified DB settings, currently SQL Mode
lcfg, err := l.cfg.Clone()
if err != nil {
return err
}
// fix nil map after clone, which we will use below
// TODO: we may develop `SafeClone` in future
if lcfg.To.Session == nil {
lcfg.To.Session = make(map[string]string)
}
timeZone := l.cfg.Timezone
if len(timeZone) == 0 {
baseDB, err2 := conn.GetDownstreamDB(&l.cfg.To)
if err2 != nil {
return err2
}
defer baseDB.Close()
var err1 error
timeZone, err1 = config.FetchTimeZoneSetting(ctx, baseDB.DB)
if err1 != nil {
return err1
}
}
lcfg.To.Session["time_zone"] = timeZone
hasSQLMode := false
for k := range l.cfg.To.Session {
if strings.ToLower(k) == "sql_mode" {
hasSQLMode = true
break
}
}
if !hasSQLMode {
sqlModes, err3 := conn.AdjustSQLModeCompatible(l.cfg.LoaderConfig.SQLMode)
if err3 != nil {
l.logger.Warn("cannot adjust sql_mode compatible, the sql_mode will stay the same", log.ShortError(err3))
}
lcfg.To.Session["sql_mode"] = sqlModes
}
l.logger.Info("loader's sql_mode is", zap.String("sqlmode", lcfg.To.Session["sql_mode"]))
l.toDB, l.toDBConns, err = createConns(tctx, lcfg, lcfg.Name, lcfg.SourceID, l.cfg.PoolSize)
if err != nil {
return err
}
return nil
}
// Process implements Unit.Process.
func (l *Loader) Process(ctx context.Context, pr chan pb.ProcessResult) {
loaderExitWithErrorCounter.WithLabelValues(l.cfg.Name, l.cfg.SourceID).Add(0)
newCtx, cancel := context.WithCancel(ctx)
defer cancel()
l.newFileJobQueue()
binlog, gtid, err := getMydumpMetadata(ctx, l.cli, l.cfg, l.workerName)
if err != nil {
loaderExitWithErrorCounter.WithLabelValues(l.cfg.Name, l.cfg.SourceID).Inc()
pr <- pb.ProcessResult{
Errors: []*pb.ProcessError{unit.NewProcessError(err)},
}
return
}
if binlog != "" {
l.metaBinlog.Store(binlog)
}
if gtid != "" {
l.metaBinlogGTID.Store(gtid)
}
l.runFatalChan = make(chan *pb.ProcessError, 2*l.cfg.PoolSize)
errs := make([]*pb.ProcessError, 0, 2)
var wg sync.WaitGroup
wg.Add(1)
go func() {
defer wg.Done()
for err := range l.runFatalChan {
cancel() // cancel l.Restore
errs = append(errs, err)
}
}()
failpoint.Inject("longLoadProcess", func(val failpoint.Value) {
if sec, ok := val.(int); ok {
l.logger.Info("long loader unit", zap.Int("second", sec))
time.Sleep(time.Duration(sec) * time.Second)
}
})
err = l.Restore(newCtx)
close(l.runFatalChan) // Restore returned, all potential fatal sent to l.runFatalChan
cancel() // cancel the goroutines created in `Restore`.
failpoint.Inject("dontWaitWorkerExit", func(_ failpoint.Value) {
l.logger.Info("", zap.String("failpoint", "dontWaitWorkerExit"))
l.workerWg.Wait()
})
wg.Wait() // wait for receive all fatal from l.runFatalChan
if err != nil {
if utils.IsContextCanceledError(err) {
l.logger.Info("filter out error caused by user cancel")
} else {
loaderExitWithErrorCounter.WithLabelValues(l.cfg.Name, l.cfg.SourceID).Inc()
errs = append(errs, unit.NewProcessError(err))
}
}
isCanceled := false
select {
case <-ctx.Done():
isCanceled = true
default:
}
pr <- pb.ProcessResult{
IsCanceled: isCanceled,
Errors: errs,
}
}
func (l *Loader) newFileJobQueue() {
l.closeFileJobQueue()
l.fileJobQueue = make(chan *fileJob, jobCount)
l.fileJobQueueClosed.Store(false)
}
func (l *Loader) closeFileJobQueue() {
if l.fileJobQueueClosed.Load() {
return
}
close(l.fileJobQueue)
l.fileJobQueueClosed.Store(true)
}
// align with https://github.com/pingcap/dumpling/pull/140
// if input is malformed, return original string and print log.
func unescapePercent(input string, logger log.Logger) string {
buf := bytes.Buffer{}
buf.Grow(len(input))
i := 0
for i < len(input) {
if input[i] != '%' {
buf.WriteByte(input[i])
i++
} else {
if i+2 >= len(input) {
logger.Error("malformed filename while unescapePercent", zap.String("filename", input))
return input
}
ascii, err := hex.DecodeString(input[i+1 : i+3])
if err != nil {
logger.Error("malformed filename while unescapePercent", zap.String("filename", input))
return input
}
buf.Write(ascii)
i += 3
}
}
return buf.String()
}
func (l *Loader) skipSchemaAndTable(table *filter.Table) bool {
if filter.IsSystemSchema(table.Schema) {
return true
}
table.Schema = unescapePercent(table.Schema, l.logger)
table.Name = unescapePercent(table.Name, l.logger)
tbs := []*filter.Table{table}
tbs = l.baList.Apply(tbs)
return len(tbs) == 0
}
func (l *Loader) isClosed() bool {
return l.closed.Load()
}
// IsFreshTask implements Unit.IsFreshTask.
func (l *Loader) IsFreshTask(ctx context.Context) (bool, error) {
count, err := l.checkPoint.Count(tcontext.NewContext(ctx, l.logger))
return count == 0, err
}
// Restore begins the restore process.
func (l *Loader) Restore(ctx context.Context) error {
if err := putLoadTask(l.cli, l.cfg, l.workerName); err != nil {
return err
}
if err := l.prepare(); err != nil {
l.logger.Error("scan directory failed", zap.String("directory", l.cfg.Dir), log.ShortError(err))
return err
}
failpoint.Inject("WaitLoaderStopBeforeLoadCheckpoint", func(v failpoint.Value) {
t := v.(int)
l.logger.Info("wait loader stop before load checkpoint")
l.wg.Add(1)
time.Sleep(time.Duration(t) * time.Second)
l.wg.Done()
})
// not update checkpoint in memory when restoring, so when re-Restore, we need to load checkpoint from DB
err := l.checkPoint.Load(tcontext.NewContext(ctx, l.logger))
if err != nil {
return err
}
err = l.checkPoint.CalcProgress(l.db2Tables)
if err != nil {
l.logger.Error("calc load process", log.ShortError(err))
return err
}
l.loadFinishedSize()
if err2 := l.initAndStartWorkerPool(ctx); err2 != nil {
l.logger.Error("initial and start worker pools failed", log.ShortError(err))
return err2
}
begin := time.Now()
err = l.restoreData(ctx)
failpoint.Inject("dontWaitWorkerExit", func(_ failpoint.Value) {
l.logger.Info("", zap.String("failpoint", "dontWaitWorkerExit"))
failpoint.Return(nil)
})
// make sure all workers exit
l.closeFileJobQueue() // all data file dispatched, close it
l.workerWg.Wait()
if err == nil {
l.finish.Store(true)
l.logger.Info("all data files have been finished", zap.Duration("cost time", time.Since(begin)))
if l.checkPoint.AllFinished() {
if l.cfg.Mode == config.ModeFull {
if err = delLoadTask(l.cli, l.cfg, l.workerName); err != nil {
return err
}
}
if l.cfg.CleanDumpFile {
cleanDumpFiles(ctx, l.cfg)
}
}
} else if errors.Cause(err) != context.Canceled {
return err
}
return nil
}
func (l *Loader) loadFinishedSize() {
results := l.checkPoint.GetAllRestoringFileInfo()
for file, pos := range results {
db, table, err := getDBAndTableFromFilename(file)
if err != nil {
l.logger.Warn("invalid db table sql file", zap.String("file", file), zap.Error(err))
continue
}
l.finishedDataSize.Add(pos[0])
l.dbTableDataFinishedSize[db][table].Add(pos[0])
}
}
// Close does graceful shutdown.
func (l *Loader) Close() {
l.Lock()
defer l.Unlock()
if l.isClosed() {
return
}
l.stopLoad()
if err := l.toDB.Close(); err != nil {
l.logger.Error("close downstream DB error", log.ShortError(err))
}
l.checkPoint.Close()
l.removeLabelValuesWithTaskInMetrics(l.cfg.Name)
l.closed.Store(true)
}
// Kill kill the loader without graceful.
func (l *Loader) Kill() {
// TODO: implement kill
l.Close()
}
// stopLoad stops loading, now it used by Close and Pause
// maybe we can refine the workflow more clear.
func (l *Loader) stopLoad() {
// before re-write workflow, simply close all job queue and job workers
// when resuming, re-create them
l.logger.Info("stop importing data process")
l.closeFileJobQueue()
l.workerWg.Wait()
l.logger.Debug("all workers have been closed")
l.wg.Wait()
l.logger.Debug("all loader's go-routines have been closed")
}
// Pause implements Unit.Pause.
func (l *Loader) Pause() {
if l.isClosed() {
l.logger.Warn("try to pause, but already closed")
return
}
l.stopLoad()
}
// Resume resumes the paused process.
func (l *Loader) Resume(ctx context.Context, pr chan pb.ProcessResult) {
if l.isClosed() {
l.logger.Warn("try to resume, but already closed")
return
}
if err := l.resetDBs(ctx); err != nil {
pr <- pb.ProcessResult{
IsCanceled: false,
Errors: []*pb.ProcessError{
unit.NewProcessError(err),
},
}
return
}
// continue the processing
l.Process(ctx, pr)
}
func (l *Loader) resetDBs(ctx context.Context) error {
var err error
tctx := tcontext.NewContext(ctx, l.logger)
for i := 0; i < len(l.toDBConns); i++ {
err = l.toDBConns[i].resetConn(tctx)
if err != nil {
return terror.WithScope(err, terror.ScopeDownstream)
}
}
err = l.checkPoint.ResetConn(tctx)
if err != nil {
return terror.WithScope(err, terror.ScopeDownstream)
}
return nil
}
// Update implements Unit.Update
// now, only support to update config for routes, filters, column-mappings, block-allow-list
// now no config diff implemented, so simply re-init use new config
// no binlog filter for loader need to update.
func (l *Loader) Update(ctx context.Context, cfg *config.SubTaskConfig) error {
var (
err error
oldBaList *filter.Filter
oldTableRouter *regexprrouter.RouteTable
oldColumnMapping *cm.Mapping
)
defer func() {
if err == nil {
return
}
if oldBaList != nil {
l.baList = oldBaList
}
if oldTableRouter != nil {
l.tableRouter = oldTableRouter
}
if oldColumnMapping != nil {
l.columnMapping = oldColumnMapping
}
}()
// update block-allow-list
oldBaList = l.baList
l.baList, err = filter.New(cfg.CaseSensitive, cfg.BAList)
if err != nil {
return terror.ErrLoadUnitGenBAList.Delegate(err)
}
// update route, for loader, this almost useless, because schemas often have been restored
oldTableRouter = l.tableRouter
l.tableRouter, err = regexprrouter.NewRegExprRouter(cfg.CaseSensitive, cfg.RouteRules)
if err != nil {
return terror.ErrLoadUnitGenTableRouter.Delegate(err)
}
// update column-mappings
oldColumnMapping = l.columnMapping
l.columnMapping, err = cm.NewMapping(cfg.CaseSensitive, cfg.ColumnMappingRules)
if err != nil {
return terror.ErrLoadUnitGenColumnMapping.Delegate(err)
}
// update l.cfg
l.cfg.BAList = cfg.BAList
l.cfg.RouteRules = cfg.RouteRules
l.cfg.ColumnMappingRules = cfg.ColumnMappingRules
return nil
}
func (l *Loader) genRouter(rules []*router.TableRule) error {
l.tableRouter, _ = regexprrouter.NewRegExprRouter(l.cfg.CaseSensitive, []*router.TableRule{})
for _, rule := range rules {
err := l.tableRouter.AddRule(rule)
if err != nil {
return terror.ErrLoadUnitGenTableRouter.Delegate(err)
}
}
schemaRules, tableRules := l.tableRouter.AllRules()
l.logger.Debug("all route rules", zap.Reflect("schema route rules", schemaRules), zap.Reflect("table route rules", tableRules))
return nil
}
func (l *Loader) initAndStartWorkerPool(ctx context.Context) error {
for i := 0; i < l.cfg.PoolSize; i++ {
worker := NewWorker(l, i)
l.workerWg.Add(1) // for every worker goroutine, Add(1)
go func() {
defer l.workerWg.Done()
worker.run(ctx, l.fileJobQueue, l.runFatalChan)
}()
}
return nil
}
func (l *Loader) prepareDBFiles(files map[string]struct{}) error {
// reset some variables
l.db2Tables = make(map[string]Tables2DataFiles)
l.totalFileCount.Store(0) // reset
schemaFileCount := 0
for file := range files {
db, ok := utils.GetDBFromDumpFilename(file)
if !ok {
continue
}
schemaFileCount++
if l.skipSchemaAndTable(&filter.Table{Schema: db}) {
l.logger.Warn("ignore schema file", zap.String("schema file", file))
continue
}
l.db2Tables[db] = make(Tables2DataFiles)
l.totalFileCount.Add(1) // for schema
}
if schemaFileCount == 0 {
l.logger.Warn("invalid mydumper files for there are no `-schema-create.sql` files found, and will generate later")
}
if len(l.db2Tables) == 0 {
l.logger.Warn("no available `-schema-create.sql` files, check mydumper parameter matches block-allow-list in task config, will generate later")
}
return nil
}
func (l *Loader) prepareTableFiles(files map[string]struct{}) error {