Skip to content

Commit

Permalink
ddl: limit the count of getting ddlhistory jobs (#55590) (#56141)
Browse files Browse the repository at this point in the history
close #55711
  • Loading branch information
ti-chi-bot authored Sep 24, 2024
1 parent ed63cd2 commit a64e23d
Show file tree
Hide file tree
Showing 6 changed files with 163 additions and 11 deletions.
4 changes: 2 additions & 2 deletions docs/tidb_http_api.md
Original file line number Diff line number Diff line change
Expand Up @@ -458,12 +458,12 @@ timezone.*

**Note**: If you request a TiDB that is not ddl owner, the response will be `This node is not a ddl owner, can't be resigned.`
1. Get all TiDB DDL job history information.
1. Get the TiDB DDL job history information.
```shell
curl http://{TiDBIP}:10080/ddl/history
```
**Note**: When the DDL history is very very long, it may consume a lot memory and even cause OOM. Consider adding `start_job_id` and `limit`.
**Note**: When the DDL history is very very long, system table may containg too many jobs. This interface will get a maximum of 2048 history ddl jobs by default. If you want get more jobs, consider adding `start_job_id` and `limit`.
1. Get count {number} TiDB DDL job history information.
Expand Down
1 change: 1 addition & 0 deletions pkg/ddl/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -201,6 +201,7 @@ go_test(
"ddl_algorithm_test.go",
"ddl_api_test.go",
"ddl_error_test.go",
"ddl_history_test.go",
"ddl_running_jobs_test.go",
"ddl_test.go",
"ddl_worker_test.go",
Expand Down
18 changes: 18 additions & 0 deletions pkg/ddl/ddl.go
Original file line number Diff line number Diff line change
Expand Up @@ -1708,6 +1708,9 @@ const DefNumHistoryJobs = 10

const batchNumHistoryJobs = 128

// DefNumGetDDLHistoryJobs is the max count for getting the ddl history once.
const DefNumGetDDLHistoryJobs = 2048

// GetLastNHistoryDDLJobs returns the DDL history jobs and an error.
// The maximum count of history jobs is num.
func GetLastNHistoryDDLJobs(t *meta.Meta, maxNumJobs int) ([]*model.Job, error) {
Expand Down Expand Up @@ -1788,9 +1791,23 @@ func GetAllHistoryDDLJobs(m *meta.Meta) ([]*model.Job, error) {
func ScanHistoryDDLJobs(m *meta.Meta, startJobID int64, limit int) ([]*model.Job, error) {
var iter meta.LastJobIterator
var err error

if startJobID == 0 {
// if 'start_job_id' == 0 and 'limit' == 0(default value), get the last 1024 ddl history job by defaultly.
if limit == 0 {
limit = DefNumGetDDLHistoryJobs

failpoint.Inject("history-ddl-jobs-limit", func(val failpoint.Value) {
injectLimit, ok := val.(int)
if ok {
logutil.BgLogger().Info("failpoint history-ddl-jobs-limit", zap.Int("limit", injectLimit))
limit = injectLimit
}
})
}
iter, err = m.GetLastHistoryDDLJobsIterator()
} else {
// if 'start_job_id' > 0, it must set value to 'limit'
if limit == 0 {
return nil, errors.New("when 'start_job_id' is specified, it must work with a 'limit'")
}
Expand All @@ -1799,6 +1816,7 @@ func ScanHistoryDDLJobs(m *meta.Meta, startJobID int64, limit int) ([]*model.Job
if err != nil {
return nil, errors.Trace(err)
}

return iter.GetLastJobs(limit, nil)
}

Expand Down
126 changes: 126 additions & 0 deletions pkg/ddl/ddl_history_test.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,126 @@
// Copyright 2024 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,
// 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 ddl_test

import (
"context"
"testing"

"github.com/ngaut/pools"
"github.com/pingcap/failpoint"
"github.com/pingcap/tidb/pkg/ddl"
"github.com/pingcap/tidb/pkg/ddl/internal/session"
"github.com/pingcap/tidb/pkg/kv"
"github.com/pingcap/tidb/pkg/meta"
"github.com/pingcap/tidb/pkg/parser/model"
"github.com/pingcap/tidb/pkg/testkit"
"github.com/stretchr/testify/require"
)

func TestDDLHistoryBasic(t *testing.T) {
var (
ddlHistoryJobCount = 0
)

store := testkit.CreateMockStore(t)
rs := pools.NewResourcePool(func() (pools.Resource, error) {
newTk := testkit.NewTestKit(t, store)
return newTk.Session(), nil
}, 8, 8, 0)
sessPool := session.NewSessionPool(rs, store)
sessCtx, err := sessPool.Get()
require.NoError(t, err)
sess := session.NewSession(sessCtx)
ctx := kv.WithInternalSourceType(context.Background(), kv.InternalTxnLightning)
err = kv.RunInNewTxn(ctx, store, false, func(ctx context.Context, txn kv.Transaction) error {
t := meta.NewMeta(txn)
return ddl.AddHistoryDDLJob(sess, t, &model.Job{
ID: 1,
}, false)
})
require.NoError(t, err)
err = kv.RunInNewTxn(ctx, store, false, func(ctx context.Context, txn kv.Transaction) error {
t := meta.NewMeta(txn)
return ddl.AddHistoryDDLJob(sess, t, &model.Job{
ID: 2,
}, false)
})
require.NoError(t, err)
job, err := ddl.GetHistoryJobByID(sessCtx, 1)
require.NoError(t, err)
require.Equal(t, int64(1), job.ID)
err = kv.RunInNewTxn(ctx, store, false, func(ctx context.Context, txn kv.Transaction) error {
m := meta.NewMeta(txn)
jobs, err := ddl.GetLastNHistoryDDLJobs(m, 2)
require.NoError(t, err)
require.Equal(t, 2, len(jobs))
return nil
})
require.NoError(t, err)

err = kv.RunInNewTxn(ctx, store, false, func(ctx context.Context, txn kv.Transaction) error {
m := meta.NewMeta(txn)
jobs, err := ddl.GetAllHistoryDDLJobs(m)
require.NoError(t, err)
ddlHistoryJobCount = len(jobs)
return nil
})

require.NoError(t, err)
err = kv.RunInNewTxn(ctx, store, false, func(ctx context.Context, txn kv.Transaction) error {
m := meta.NewMeta(txn)
jobs, err := ddl.ScanHistoryDDLJobs(m, 2, 2)
require.NoError(t, err)
require.Equal(t, 2, len(jobs))
require.Equal(t, int64(2), jobs[0].ID)
require.Equal(t, int64(1), jobs[1].ID)
return nil
})

require.NoError(t, err)

require.NoError(t, failpoint.Enable("github.com/pingcap/tidb/pkg/ddl/history-ddl-jobs-limit", "return(128)"))
defer func() {
require.NoError(t, failpoint.Disable("github.com/pingcap/tidb/pkg/ddl/history-ddl-jobs-limit"))
}()

err = kv.RunInNewTxn(ctx, store, false, func(ctx context.Context, txn kv.Transaction) error {
m := meta.NewMeta(txn)
jobs, err := ddl.ScanHistoryDDLJobs(m, 0, 0)
require.NoError(t, err)
if ddlHistoryJobCount <= 128 {
require.Equal(t, ddlHistoryJobCount, len(jobs))
} else {
require.Equal(t, 128, len(jobs))
}
require.True(t, len(jobs) > 2)
require.Equal(t, int64(2), jobs[ddlHistoryJobCount-2].ID)
require.Equal(t, int64(1), jobs[ddlHistoryJobCount-1].ID)
return nil
})

require.NoError(t, err)
}

func TestScanHistoryDDLJobsWithErrorLimit(t *testing.T) {
var (
m = &meta.Meta{}
startJobID int64 = 10
limit = 0
)

_, err := ddl.ScanHistoryDDLJobs(m, startJobID, limit)
require.ErrorContains(t, err, "when 'start_job_id' is specified, it must work with a 'limit'")
}
7 changes: 7 additions & 0 deletions pkg/server/handler/tests/http_handler_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -16,6 +16,7 @@ package tests

import (
"bytes"
"cmp"
"context"
"crypto/tls"
"crypto/x509"
Expand All @@ -29,6 +30,7 @@ import (
"net/http"
"net/http/httptest"
"net/http/httputil"
"slices"
"sort"
"testing"
"time"
Expand Down Expand Up @@ -992,6 +994,11 @@ func TestAllHistory(t *testing.T) {
data, err := ddl.GetAllHistoryDDLJobs(txnMeta)
require.NoError(t, err)
err = decoder.Decode(&jobs)
require.True(t, len(jobs) < ddl.DefNumGetDDLHistoryJobs)
// sort job.
slices.SortFunc(jobs, func(i, j *model.Job) int {
return cmp.Compare(i.ID, j.ID)
})

require.NoError(t, err)
require.NoError(t, resp.Body.Close())
Expand Down
18 changes: 9 additions & 9 deletions pkg/server/handler/tikvhandler/tikv_handler.go
Original file line number Diff line number Diff line change
Expand Up @@ -1089,8 +1089,11 @@ func (h *TableHandler) ServeHTTP(w http.ResponseWriter, req *http.Request) {

// ServeHTTP handles request of ddl jobs history.
func (h DDLHistoryJobHandler) ServeHTTP(w http.ResponseWriter, req *http.Request) {
var jobID, limitID int
var err error
var (
jobID = 0
limitID = 0
err error
)
if jobValue := req.FormValue(handler.JobID); len(jobValue) > 0 {
jobID, err = strconv.Atoi(jobValue)
if err != nil {
Expand All @@ -1108,8 +1111,9 @@ func (h DDLHistoryJobHandler) ServeHTTP(w http.ResponseWriter, req *http.Request
writeError(w, err)
return
}
if limitID < 1 {
writeError(w, errors.New("ddl history limit must be greater than 0"))
if limitID < 1 || limitID > ddl.DefNumGetDDLHistoryJobs {
handler.WriteError(w,
errors.Errorf("ddl history limit must be greater than 0 and less than or equal to %v", ddl.DefNumGetDDLHistoryJobs))
return
}
}
Expand All @@ -1129,11 +1133,7 @@ func (h DDLHistoryJobHandler) getHistoryDDL(jobID, limit int) (jobs []*model.Job
}
txnMeta := meta.NewMeta(txn)

if jobID == 0 && limit == 0 {
jobs, err = ddl.GetAllHistoryDDLJobs(txnMeta)
} else {
jobs, err = ddl.ScanHistoryDDLJobs(txnMeta, int64(jobID), limit)
}
jobs, err = ddl.ScanHistoryDDLJobs(txnMeta, int64(jobID), limit)
if err != nil {
return nil, errors.Trace(err)
}
Expand Down

0 comments on commit a64e23d

Please sign in to comment.