Skip to content

Commit 9411e82

Browse files
authored
statistics: use goroutine pool to improve performance (#46266)
close #46267
1 parent 94cfa8b commit 9411e82

File tree

5 files changed

+23
-10
lines changed

5 files changed

+23
-10
lines changed

go.mod

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -259,7 +259,7 @@ require (
259259
github.com/spaolacci/murmur3 v1.1.0 // indirect
260260
github.com/spf13/cast v1.5.0 // indirect
261261
github.com/stretchr/objx v0.5.0 // indirect
262-
github.com/tiancaiamao/gp v0.0.0-20221230034425-4025bc8a4d4a // indirect
262+
github.com/tiancaiamao/gp v0.0.0-20221230034425-4025bc8a4d4a
263263
github.com/tklauser/go-sysconf v0.3.11 // indirect
264264
github.com/tklauser/numcpus v0.6.0 // indirect
265265
github.com/tmc/grpc-websocket-proxy v0.0.0-20201229170055-e5319fda7802 // indirect

statistics/BUILD.bazel

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -121,6 +121,7 @@ go_test(
121121
"@com_github_pingcap_failpoint//:failpoint",
122122
"@com_github_pingcap_log//:log",
123123
"@com_github_stretchr_testify//require",
124+
"@com_github_tiancaiamao_gp//:gp",
124125
"@org_uber_go_goleak//:goleak",
125126
"@org_uber_go_zap//:zap",
126127
],

statistics/cmsketch_bench_test.go

Lines changed: 5 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -27,6 +27,7 @@ import (
2727
"github.com/pingcap/tidb/util/chunk"
2828
"github.com/pingcap/tidb/util/codec"
2929
"github.com/stretchr/testify/require"
30+
"github.com/tiancaiamao/gp"
3031
)
3132

3233
// cmd: go test -run=^$ -bench=BenchmarkMergePartTopN2GlobalTopNWithHists -benchmem github.com/pingcap/tidb/statistics
@@ -131,15 +132,17 @@ func benchmarkMergeGlobalStatsTopNByConcurrencyWithHists(partitions int, b *test
131132
} else if batchSize > handle.MaxPartitionMergeBatchSize {
132133
batchSize = handle.MaxPartitionMergeBatchSize
133134
}
135+
gpool := gp.New(mergeConcurrency, 5*time.Minute)
136+
defer gpool.Close()
134137
b.ResetTimer()
135138
for i := 0; i < b.N; i++ {
136139
// Benchmark merge 10 topN.
137-
_, _, _, _ = handle.MergeGlobalStatsTopNByConcurrency(mergeConcurrency, batchSize, wrapper, loc, version, 10, false, &isKilled)
140+
_, _, _, _ = handle.MergeGlobalStatsTopNByConcurrency(gpool, mergeConcurrency, batchSize, wrapper, loc, version, 10, false, &isKilled)
138141
}
139142
}
140143

141144
var benchmarkSizes = []int{100, 1000, 10000, 100000, 1000000, 10000000}
142-
var benchmarkConcurrencySizes = []int{100, 1000, 10000, 100000, 1000000, 10000000, 100000000}
145+
var benchmarkConcurrencySizes = []int{100, 1000, 10000, 100000}
143146

144147
func BenchmarkMergePartTopN2GlobalTopNWithHists(b *testing.B) {
145148
for _, size := range benchmarkSizes {

statistics/handle/BUILD.bazel

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -48,6 +48,7 @@ go_library(
4848
"@com_github_pingcap_failpoint//:failpoint",
4949
"@com_github_pingcap_log//:log",
5050
"@com_github_pingcap_tipb//go-tipb",
51+
"@com_github_tiancaiamao_gp//:gp",
5152
"@com_github_tikv_client_go_v2//oracle",
5253
"@org_uber_go_atomic//:atomic",
5354
"@org_uber_go_zap//:zap",

statistics/handle/handle.go

Lines changed: 15 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -18,6 +18,7 @@ import (
1818
"context"
1919
"encoding/json"
2020
"fmt"
21+
"math"
2122
"slices"
2223
"strconv"
2324
"strings"
@@ -41,12 +42,12 @@ import (
4142
handle_metrics "github.com/pingcap/tidb/statistics/handle/metrics"
4243
"github.com/pingcap/tidb/table"
4344
"github.com/pingcap/tidb/types"
44-
"github.com/pingcap/tidb/util"
4545
"github.com/pingcap/tidb/util/chunk"
4646
"github.com/pingcap/tidb/util/logutil"
4747
"github.com/pingcap/tidb/util/mathutil"
4848
"github.com/pingcap/tidb/util/sqlexec"
4949
"github.com/pingcap/tidb/util/syncutil"
50+
"github.com/tiancaiamao/gp"
5051
"github.com/tikv/client-go/v2/oracle"
5152
atomic2 "go.uber.org/atomic"
5253
"go.uber.org/zap"
@@ -62,6 +63,9 @@ const (
6263

6364
// Handle can update stats info periodically.
6465
type Handle struct {
66+
// this gpool is used to reuse goroutine in the mergeGlobalStatsTopN.
67+
gpool *gp.Pool
68+
6569
pool sessionPool
6670

6771
// initStatsCtx is the ctx only used for initStats
@@ -483,6 +487,7 @@ type sessionPool interface {
483487
func NewHandle(ctx, initStatsCtx sessionctx.Context, lease time.Duration, pool sessionPool, tracker sessionctx.SysProcTracker, autoAnalyzeProcIDGetter func() uint64) (*Handle, error) {
484488
cfg := config.GetGlobalConfig()
485489
handle := &Handle{
490+
gpool: gp.New(math.MaxInt16, time.Minute),
486491
ddlEventCh: make(chan *ddlUtil.Event, 1000),
487492
listHead: &SessionStatsCollector{mapper: make(tableDeltaMap), rateMap: make(errorRateDeltaMap)},
488493
idxUsageListHead: &SessionIndexUsageCollector{mapper: make(indexUsageMap)},
@@ -857,7 +862,7 @@ func (h *Handle) mergePartitionStats2GlobalStats(sc sessionctx.Context,
857862
// These remaining topN numbers will be used as a separate bucket for later histogram merging.
858863
var popedTopN []statistics.TopNMeta
859864
wrapper := statistics.NewStatsWrapper(allHg[i], allTopN[i])
860-
globalStats.TopN[i], popedTopN, allHg[i], err = mergeGlobalStatsTopN(sc, wrapper, sc.GetSessionVars().StmtCtx.TimeZone, sc.GetSessionVars().AnalyzeVersion, uint32(opts[ast.AnalyzeOptNumTopN]), isIndex == 1)
865+
globalStats.TopN[i], popedTopN, allHg[i], err = mergeGlobalStatsTopN(h.gpool, sc, wrapper, sc.GetSessionVars().StmtCtx.TimeZone, sc.GetSessionVars().AnalyzeVersion, uint32(opts[ast.AnalyzeOptNumTopN]), isIndex == 1)
861866
if err != nil {
862867
return
863868
}
@@ -889,7 +894,7 @@ func (h *Handle) mergePartitionStats2GlobalStats(sc sessionctx.Context,
889894
return
890895
}
891896

892-
func mergeGlobalStatsTopN(sc sessionctx.Context, wrapper *statistics.StatsWrapper,
897+
func mergeGlobalStatsTopN(gp *gp.Pool, sc sessionctx.Context, wrapper *statistics.StatsWrapper,
893898
timeZone *time.Location, version int, n uint32, isIndex bool) (*statistics.TopN,
894899
[]statistics.TopNMeta, []*statistics.Histogram, error) {
895900
mergeConcurrency := sc.GetSessionVars().AnalyzePartitionMergeConcurrency
@@ -904,14 +909,14 @@ func mergeGlobalStatsTopN(sc sessionctx.Context, wrapper *statistics.StatsWrappe
904909
} else if batchSize > MaxPartitionMergeBatchSize {
905910
batchSize = MaxPartitionMergeBatchSize
906911
}
907-
return MergeGlobalStatsTopNByConcurrency(mergeConcurrency, batchSize, wrapper, timeZone, version, n, isIndex, killed)
912+
return MergeGlobalStatsTopNByConcurrency(gp, mergeConcurrency, batchSize, wrapper, timeZone, version, n, isIndex, killed)
908913
}
909914

910915
// MergeGlobalStatsTopNByConcurrency merge partition topN by concurrency
911916
// To merge global stats topn by concurrency, we will separate the partition topn in concurrency part and deal it with different worker.
912917
// mergeConcurrency is used to control the total concurrency of the running worker, and mergeBatchSize is sued to control
913918
// the partition size for each worker to solve it
914-
func MergeGlobalStatsTopNByConcurrency(mergeConcurrency, mergeBatchSize int, wrapper *statistics.StatsWrapper,
919+
func MergeGlobalStatsTopNByConcurrency(gp *gp.Pool, mergeConcurrency, mergeBatchSize int, wrapper *statistics.StatsWrapper,
915920
timeZone *time.Location, version int, n uint32, isIndex bool, killed *uint32) (*statistics.TopN,
916921
[]statistics.TopNMeta, []*statistics.Histogram, error) {
917922
if len(wrapper.AllTopN) < mergeConcurrency {
@@ -927,13 +932,15 @@ func MergeGlobalStatsTopNByConcurrency(mergeConcurrency, mergeBatchSize int, wra
927932
tasks = append(tasks, task)
928933
start = end
929934
}
930-
var wg util.WaitGroupWrapper
935+
var wg sync.WaitGroup
931936
taskNum := len(tasks)
932937
taskCh := make(chan *statistics.TopnStatsMergeTask, taskNum)
933938
respCh := make(chan *statistics.TopnStatsMergeResponse, taskNum)
934939
for i := 0; i < mergeConcurrency; i++ {
935940
worker := statistics.NewTopnStatsMergeWorker(taskCh, respCh, wrapper, killed)
936-
wg.Run(func() {
941+
wg.Add(1)
942+
gp.Go(func() {
943+
defer wg.Done()
937944
worker.Run(timeZone, isIndex, n, version)
938945
})
939946
}
@@ -2322,4 +2329,5 @@ func (h *Handle) SetStatsCacheCapacity(c int64) {
23222329
// Close stops the background
23232330
func (h *Handle) Close() {
23242331
h.statsCache.Load().Close()
2332+
h.gpool.Close()
23252333
}

0 commit comments

Comments
 (0)