Skip to content

Commit f30a668

Browse files
craig[bot]wenyihu6
andcommitted
Merge #154688
154688: mmaprototype: pass []roachpb.StoreID for computeMeansForStoreSet r=tbg a=wenyihu6 Epic: CRDB-55052 Release note: none --- **mmaprototype: pass []roachpb.StoreID for computeMeansForStoreSet** Previously, computeMeansForStoreSet took in a meansForStoreSet struct. Future PRs will use this helper function to compute mean load summaries for stores without having a storeIDPostingList handy. To avoid unnecessary construction of a storeIDPostingList, the function signature now takes meansLoad and []roachpb.StoreID directly. This change lets future callers pass a simple slice of stores. Since future callers may provide slices with duplicate store IDs, the function now also de-duplicates them internally. --- **mmaprototype: refactor computeMeansForStoreSet** Previously, we updated computeMeansForStoreSet to take meansLoad and []roachpb.StoreID directly instead of a meansForStoreSet struct. This commit updates the call sites that only require meansLoad to use it directly, removing unnecessary use of meansForStoreSet. --- **mmaprototype: pass in scratch stores for computeMeansForStoreSet** Previously, computeMeansForStoreSet allocated a new map on every call to deduplicate the provided stores list. This commit refactors it to reuse a scratchStores map (similar to scratchNodes). The caller now allocates this map once and stores it in the function scope or struct, reducing repeated allocations. --- **mmaprototype: return meansLoad directly for computeMeansForStoreSet** Previously, computeMeansForStoreSet received *meansForStoreSet, which contained information like stores needed to compute the mean. A recent commit changed it to take *meansLoad, and computeMeansForStoreSet doesn’t actually use any fields as provided information. This commit updates computeMeansForStoreSet to directly construct and return a meanLoad instead. Co-authored-by: wenyihu6 <[email protected]>
2 parents f520554 + b68a7d8 commit f30a668

File tree

3 files changed

+35
-18
lines changed

3 files changed

+35
-18
lines changed

pkg/kv/kvserver/allocator/mmaprototype/allocator_state.go

Lines changed: 4 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -356,6 +356,7 @@ func (a *allocatorState) rebalanceStores(
356356
var storesToExclude storeIDPostingList
357357
var storesToExcludeForRange storeIDPostingList
358358
scratchNodes := map[roachpb.NodeID]*NodeLoad{}
359+
scratchStores := map[roachpb.StoreID]struct{}{}
359360
// The caller has a fixed concurrency limit it can move ranges at, when it
360361
// is the sender of the snapshot. So we don't want to create too many
361362
// changes, since then the allocator gets too far ahead of what has been
@@ -473,10 +474,8 @@ func (a *allocatorState) rebalanceStores(
473474
if len(candsPL) <= 1 {
474475
continue // leaseholder is the only candidate
475476
}
476-
var means meansForStoreSet
477477
clear(scratchNodes)
478-
means.stores = candsPL
479-
computeMeansForStoreSet(a.cs, &means, scratchNodes)
478+
means := computeMeansForStoreSet(a.cs, candsPL, scratchNodes, scratchStores)
480479
sls := a.cs.computeLoadSummary(ctx, store.StoreID, &means.storeLoad, &means.nodeLoad)
481480
log.KvDistribution.VInfof(ctx, 2, "considering lease-transfer r%v from s%v: candidates are %v", rangeID, store.StoreID, candsPL)
482481
if sls.dimSummary[CPURate] < overloadSlow {
@@ -948,7 +947,7 @@ type candidateInfo struct {
948947

949948
type candidateSet struct {
950949
candidates []candidateInfo
951-
means *meansForStoreSet
950+
means *meansLoad
952951
}
953952

954953
type ignoreLevel uint8
@@ -1390,7 +1389,7 @@ func (a *allocatorState) computeCandidatesForRange(
13901389
storeLoadSummary: csls,
13911390
})
13921391
}
1393-
cset.means = means
1392+
cset.means = &means.meansLoad
13941393
return cset, sheddingSLS
13951394
}
13961395

pkg/kv/kvserver/allocator/mmaprototype/cluster_state.go

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -1949,7 +1949,7 @@ func (cs *clusterState) canShedAndAddLoad(
19491949
srcSS *storeState,
19501950
targetSS *storeState,
19511951
delta LoadVector,
1952-
means *meansForStoreSet,
1952+
means *meansLoad,
19531953
onlyConsiderTargetCPUSummary bool,
19541954
overloadedDim LoadDimension,
19551955
) (canAddLoad bool) {

pkg/kv/kvserver/allocator/mmaprototype/load.go

Lines changed: 30 additions & 12 deletions
Original file line numberDiff line numberDiff line change
@@ -228,11 +228,14 @@ func (sls storeLoadSummary) SafeFormat(w redact.SafePrinter, _ rune) {
228228
// storeLoadSummary no longer matches that of storeState.loadSeqNum.
229229
type meansForStoreSet struct {
230230
constraintsDisj
231-
stores storeIDPostingList
231+
meansLoad
232+
stores storeIDPostingList
233+
storeSummaries map[roachpb.StoreID]storeLoadSummary
234+
}
235+
236+
type meansLoad struct {
232237
storeLoad meanStoreLoad
233238
nodeLoad meanNodeLoad
234-
235-
storeSummaries map[roachpb.StoreID]storeLoadSummary
236239
}
237240

238241
var _ mapEntry = &meansForStoreSet{}
@@ -277,7 +280,8 @@ type meansMemo struct {
277280
constraintMatcher *constraintMatcher
278281
meansMap *clearableMemoMap[constraintsDisj, *meansForStoreSet]
279282

280-
scratchNodes map[roachpb.NodeID]*NodeLoad
283+
scratchNodes map[roachpb.NodeID]*NodeLoad
284+
scratchStores map[roachpb.StoreID]struct{}
281285
}
282286

283287
var meansForStoreSetSlicePool = sync.Pool{
@@ -317,7 +321,8 @@ func newMeansMemo(
317321
constraintMatcher: constraintMatcher,
318322
meansMap: newClearableMapMemo[constraintsDisj, *meansForStoreSet](
319323
meansForStoreSetAllocator{}, meansForStoreSetSlicePoolImpl{}),
320-
scratchNodes: map[roachpb.NodeID]*NodeLoad{},
324+
scratchNodes: map[roachpb.NodeID]*NodeLoad{},
325+
scratchStores: map[roachpb.StoreID]struct{}{},
321326
}
322327
}
323328

@@ -339,7 +344,7 @@ func (mm *meansMemo) getMeans(expr constraintsDisj) *meansForStoreSet {
339344
}
340345
means.constraintsDisj = expr
341346
mm.constraintMatcher.constrainStoresForExpr(expr, &means.stores)
342-
computeMeansForStoreSet(mm.loadInfoProvider, means, mm.scratchNodes)
347+
means.meansLoad = computeMeansForStoreSet(mm.loadInfoProvider, means.stores, mm.scratchNodes, mm.scratchStores)
343348
return means
344349
}
345350

@@ -362,18 +367,30 @@ func (mm *meansMemo) getStoreLoadSummary(
362367
// It does not do any filtering e.g. the stores can include fdDead stores. It
363368
// is up to the caller to adjust means.stores if it wants to do filtering.
364369
//
370+
// stores may contain duplicate storeIDs, in which case computeMeansForStoreSet
371+
// should deduplicate processing of the stores. stores should be immutable.
372+
//
365373
// TODO: fix callers to exclude stores based on node failure detection, from
366374
// the mean.
367375
func computeMeansForStoreSet(
368-
loadProvider loadInfoProvider, means *meansForStoreSet, scratchNodes map[roachpb.NodeID]*NodeLoad,
369-
) {
370-
n := len(means.stores)
371-
if n == 0 {
372-
panic(fmt.Sprintf("no stores for meansForStoreSet: %v", *means))
376+
loadProvider loadInfoProvider,
377+
stores []roachpb.StoreID,
378+
scratchNodes map[roachpb.NodeID]*NodeLoad,
379+
scratchStores map[roachpb.StoreID]struct{},
380+
) (means meansLoad) {
381+
if len(stores) == 0 {
382+
panic(fmt.Sprintf("no stores for meansForStoreSet: %v", stores))
373383
}
374384
clear(scratchNodes)
375-
for _, storeID := range means.stores {
385+
clear(scratchStores)
386+
n := 0
387+
for _, storeID := range stores {
376388
nodeID, sload := loadProvider.getStoreReportedLoad(storeID)
389+
if _, ok := scratchStores[storeID]; ok {
390+
continue
391+
}
392+
n++
393+
scratchStores[storeID] = struct{}{}
377394
for j := range sload.reportedLoad {
378395
means.storeLoad.load[j] += sload.reportedLoad[j]
379396
if sload.capacity[j] == UnknownCapacity {
@@ -413,6 +430,7 @@ func computeMeansForStoreSet(
413430
float64(means.nodeLoad.loadCPU) / float64(means.nodeLoad.capacityCPU)
414431
means.nodeLoad.loadCPU /= LoadValue(n)
415432
means.nodeLoad.capacityCPU /= LoadValue(n)
433+
return means
416434
}
417435

418436
// loadSummary aggregates across all load dimensions for a store, or a node.

0 commit comments

Comments
 (0)