Skip to content

Commit 30698ad

Browse files
ti-chi-bothawkingrei
authored andcommitted
statistics: upgrade stats timeout checkpoint after it timeouts (pingcap#52424) (pingcap#52439)
close pingcap#52425
1 parent e011cdd commit 30698ad

File tree

3 files changed

+13
-8
lines changed

3 files changed

+13
-8
lines changed

statistics/handle/BUILD.bazel

+1
Original file line numberDiff line numberDiff line change
@@ -80,6 +80,7 @@ go_test(
8080
"//domain",
8181
"//parser/model",
8282
"//parser/mysql",
83+
"//sessionctx",
8384
"//sessionctx/stmtctx",
8485
"//sessionctx/variable",
8586
"//statistics",

statistics/handle/handle_hist.go

+5-4
Original file line numberDiff line numberDiff line change
@@ -214,7 +214,7 @@ func (h *Handle) SubLoadWorker(ctx sessionctx.Context, exit chan struct{}, exitW
214214
// if the last task is not successfully handled in last round for error or panic, pass it to this round to retry
215215
var lastTask *NeededItemTask
216216
for {
217-
task, err := h.HandleOneTask(lastTask, readerCtx, ctx.(sqlexec.RestrictedSQLExecutor), exit)
217+
task, err := h.HandleOneTask(ctx, lastTask, readerCtx, ctx.(sqlexec.RestrictedSQLExecutor), exit)
218218
lastTask = task
219219
if err != nil {
220220
switch err {
@@ -235,7 +235,7 @@ func (h *Handle) SubLoadWorker(ctx sessionctx.Context, exit chan struct{}, exitW
235235
// - If the task is handled successfully, return nil, nil.
236236
// - If the task is timeout, return the task and nil. The caller should retry the timeout task without sleep.
237237
// - If the task is failed, return the task, error. The caller should retry the timeout task with sleep.
238-
func (h *Handle) HandleOneTask(lastTask *NeededItemTask, readerCtx *StatsReaderContext, ctx sqlexec.RestrictedSQLExecutor, exit chan struct{}) (task *NeededItemTask, err error) {
238+
func (h *Handle) HandleOneTask(sctx sessionctx.Context, lastTask *NeededItemTask, readerCtx *StatsReaderContext, ctx sqlexec.RestrictedSQLExecutor, exit chan struct{}) (task *NeededItemTask, err error) {
239239
defer func() {
240240
// recover for each task, worker keeps working
241241
if r := recover(); r != nil {
@@ -244,7 +244,7 @@ func (h *Handle) HandleOneTask(lastTask *NeededItemTask, readerCtx *StatsReaderC
244244
}
245245
}()
246246
if lastTask == nil {
247-
task, err = h.drainColTask(exit)
247+
task, err = h.drainColTask(sctx, exit)
248248
if err != nil {
249249
if err != errExit {
250250
logutil.BgLogger().Error("Fail to drain task for stats loading.", zap.Error(err))
@@ -447,7 +447,7 @@ func (h *Handle) readStatsForOneItem(item model.TableItemID, w *statsWrapper, re
447447
}
448448

449449
// drainColTask will hang until a column task can return, and either task or error will be returned.
450-
func (h *Handle) drainColTask(exit chan struct{}) (*NeededItemTask, error) {
450+
func (h *Handle) drainColTask(sctx sessionctx.Context, exit chan struct{}) (*NeededItemTask, error) {
451451
// select NeededColumnsCh firstly, if no task, then select TimeoutColumnsCh
452452
for {
453453
select {
@@ -460,6 +460,7 @@ func (h *Handle) drainColTask(exit chan struct{}) (*NeededItemTask, error) {
460460
// if the task has already timeout, no sql is sync-waiting for it,
461461
// so do not handle it just now, put it to another channel with lower priority
462462
if time.Now().After(task.ToTimeout) {
463+
task.ToTimeout.Add(time.Duration(sctx.GetSessionVars().StatsLoadSyncWait.Load()) * time.Microsecond)
463464
h.writeToTimeoutChan(h.StatsLoad.TimeoutItemsCh, task)
464465
continue
465466
}

statistics/handle/handle_hist_test.go

+7-4
Original file line numberDiff line numberDiff line change
@@ -21,6 +21,7 @@ import (
2121
"github.com/pingcap/failpoint"
2222
"github.com/pingcap/tidb/config"
2323
"github.com/pingcap/tidb/parser/model"
24+
"github.com/pingcap/tidb/sessionctx"
2425
"github.com/pingcap/tidb/sessionctx/stmtctx"
2526
"github.com/pingcap/tidb/statistics/handle"
2627
"github.com/pingcap/tidb/testkit"
@@ -205,7 +206,7 @@ func TestConcurrentLoadHistWithPanicAndFail(t *testing.T) {
205206
exitCh := make(chan struct{})
206207
require.NoError(t, failpoint.Enable(fp.failPath, fp.inTerms))
207208

208-
task1, err1 := h.HandleOneTask(nil, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
209+
task1, err1 := h.HandleOneTask(testKit.Session().(sessionctx.Context), nil, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
209210
require.Error(t, err1)
210211
require.NotNil(t, task1)
211212
for _, resultCh := range stmtCtx1.StatsLoad.ResultCh {
@@ -226,7 +227,7 @@ func TestConcurrentLoadHistWithPanicAndFail(t *testing.T) {
226227
}
227228

228229
require.NoError(t, failpoint.Disable(fp.failPath))
229-
task3, err3 := h.HandleOneTask(task1, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
230+
task3, err3 := h.HandleOneTask(testKit.Session().(sessionctx.Context), task1, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
230231
require.NoError(t, err3)
231232
require.Nil(t, task3)
232233

@@ -305,7 +306,8 @@ func TestRetry(t *testing.T) {
305306
)
306307
readerCtx := &handle.StatsReaderContext{}
307308
for i := 0; i < handle.RetryCount; i++ {
308-
task1, err1 = h.HandleOneTask(task1, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
309+
task1, err1 = h.HandleOneTask(testKit.Session().(sessionctx.Context),
310+
task1, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
309311
require.Error(t, err1)
310312
require.NotNil(t, task1)
311313
select {
@@ -315,7 +317,8 @@ func TestRetry(t *testing.T) {
315317
default:
316318
}
317319
}
318-
result, err1 := h.HandleOneTask(task1, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
320+
result, err1 := h.HandleOneTask(testKit.Session().(sessionctx.Context),
321+
task1, readerCtx, testKit.Session().(sqlexec.RestrictedSQLExecutor), exitCh)
319322
require.NoError(t, err1)
320323
require.Nil(t, result)
321324
for _, resultCh := range stmtCtx1.StatsLoad.ResultCh {

0 commit comments

Comments
 (0)