Skip to content

Commit

Permalink
storage: rework RunGC so it no longer buffers keys and values in memory
Browse files Browse the repository at this point in the history
This commit reworks the processing of replicated state underneath the gcQueue
for the purpose of determining and sending GC requests. The primary intention
of this commit is to remove the need to buffer all of the versions of a key
in memory. As we learned in #42531, this bufferring can be extremely
unfortunate when using sequence data types which are written to frequently.

Prior to this commit, the code forward iterates through the range's data and
eagerly reads all versions of the a key into memory. It then binary searches
those versions to find the latest timestamp for the key which can be GC'd.
It then reverse iterates through all of those versions to determine the latest
version of the key which would put the current batch over its limit. This last
step works to paginate the process of actually deleting the data for many
versions of the same key. I suppose this pagination was added to ensure that
write batches due to GC requests don't get too large. Unfortunately this logic
was unable to paginate the loading of versions from the storage engine.

In this new commit, the entire process of computing data to GC now uses reverse
iteration; for each key we examine versions from oldest to newest. The commit
adds a `gcIterator` which wraps this reverse iteration with some useful
lookahead. During this GC process, at most two additional versions need to
examined to determine whether a given version is garbage.

While this approach relies on reverse iteration which is known to be less
efficient than forward iteration, it offers the opportunity to avoid allocating
memory for versions of a key which are not going to end up as a part of a GC
request. This reduction in memory usage shows up in benchmarks (see below).
The change retains the old implementation as a testing strategy and as a basis
for the benchmarks.

```
name                                                                                                                      old time/op    new time/op    delta
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000-8                  924ns ± 8%     590ns ± 1%   -36.13%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#01-8               976ns ± 2%     578ns ± 1%   -40.75%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#02-8               944ns ± 0%     570ns ± 9%   -39.63%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#03-8               903ns ± 0%     612ns ± 3%   -32.29%  (p=0.016 n=4+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#04-8               994ns ± 9%     592ns ± 9%   -40.47%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000-8               669ns ± 4%     526ns ± 1%   -21.34%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#01-8            624ns ± 0%     529ns ± 2%   -15.16%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#02-8            636ns ± 4%     534ns ± 2%   -16.04%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#03-8            612ns ± 1%     532ns ± 3%   -13.08%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#04-8            638ns ± 2%     534ns ±10%   -16.35%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000-8        603ns ± 6%     527ns ± 8%   -12.51%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#01-8     613ns ± 5%     517ns ± 6%   -15.78%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#02-8     619ns ± 6%     534ns ± 4%   -13.61%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#03-8     607ns ± 7%     520ns ± 2%   -14.39%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#04-8     599ns ± 4%     501ns ± 7%   -16.36%  (p=0.008 n=5+5)

name                                                                                                                      old speed      new speed      delta
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000-8               23.9MB/s ± 8%  37.3MB/s ± 1%   +56.23%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#01-8            22.6MB/s ± 2%  38.1MB/s ± 1%   +68.81%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#02-8            23.3MB/s ± 0%  38.7MB/s ± 9%   +66.06%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#03-8            24.4MB/s ± 0%  36.0MB/s ± 3%   +47.73%  (p=0.016 n=4+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#04-8            22.2MB/s ± 8%  37.3MB/s ± 9%   +68.09%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000-8            34.4MB/s ± 4%  43.7MB/s ± 1%   +27.08%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#01-8         36.9MB/s ± 0%  43.4MB/s ± 2%   +17.84%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#02-8         36.2MB/s ± 4%  43.1MB/s ± 2%   +19.02%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#03-8         37.6MB/s ± 1%  43.3MB/s ± 3%   +15.02%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#04-8         36.0MB/s ± 2%  43.2MB/s ±10%   +19.87%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000-8     36.5MB/s ± 5%  41.8MB/s ± 9%   +14.39%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#01-8  35.9MB/s ± 5%  42.7MB/s ± 6%   +18.83%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#02-8  35.6MB/s ± 6%  41.2MB/s ± 4%   +15.66%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#03-8  36.3MB/s ± 6%  42.3MB/s ± 2%   +16.69%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#04-8  36.7MB/s ± 4%  44.0MB/s ± 7%   +19.69%  (p=0.008 n=5+5)

name                                                                                                                      old alloc/op   new alloc/op   delta
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000-8                   325B ± 0%       76B ± 0%   -76.62%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#01-8                358B ± 0%       49B ± 0%      ~     (p=0.079 n=4+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#02-8                340B ± 0%       29B ± 0%   -91.47%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#03-8                328B ± 0%       18B ± 0%   -94.51%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#04-8                325B ± 0%       14B ± 0%   -95.69%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000-8                226B ± 0%        2B ± 0%      ~     (p=0.079 n=4+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#01-8             228B ± 0%        3B ± 0%   -98.69%  (p=0.000 n=5+4)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#02-8             228B ± 0%        2B ± 0%   -99.12%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#03-8             228B ± 0%        2B ± 0%   -99.12%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#04-8             226B ± 0%        0B       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000-8         388B ± 2%        0B       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#01-8      391B ± 2%        0B       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#02-8      389B ± 1%        0B       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#03-8      391B ± 2%        0B       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#04-8      390B ± 1%        0B       -100.00%  (p=0.008 n=5+5)

name                                                                                                                      old allocs/op  new allocs/op  delta
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000-8                   4.00 ± 0%      0.00       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#01-8                4.00 ± 0%      0.00       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#02-8                4.00 ± 0%      0.00       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#03-8                4.00 ± 0%      0.00       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[2,3],valueLen=[1,1],keysPerValue=[1,2],deleteFrac=0.000000,intentFrac=0.100000#04-8                4.00 ± 0%      0.00       -100.00%  (p=0.008 n=5+5)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000-8                0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#01-8             0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#02-8             0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#03-8             0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1,100],deleteFrac=0.100000,intentFrac=0.100000#04-8             0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000-8         0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#01-8      0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#02-8      0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#03-8      0.00           0.00           ~     (all equal)
Run/ts=[0,100],keySuffix=[8,8],valueLen=[8,16],keysPerValue=[1000,1000000],deleteFrac=0.100000,intentFrac=0.100000#04-8      0.00           0.00           ~     (all equal)
```

Release note (bug fix): The GC process was improved to paginate the key
versions of a key to fix OOM crashes which can occur when there are
extremely large numbers of versions for a given key.
  • Loading branch information
ajwerner committed Jan 16, 2020
1 parent 0f1f133 commit 55b4936
Show file tree
Hide file tree
Showing 12 changed files with 1,683 additions and 610 deletions.
2 changes: 1 addition & 1 deletion pkg/cli/debug.go
Expand Up @@ -556,7 +556,7 @@ func runDebugGCCmd(cmd *cobra.Command, args []string) error {
for _, desc := range descs {
snap := db.NewSnapshot()
defer snap.Close()
info, err := gc.RunGC(
info, err := gc.Run(
context.Background(),
&desc,
snap,
Expand Down
254 changes: 254 additions & 0 deletions pkg/storage/gc/data_distribution_test.go
@@ -0,0 +1,254 @@
// Copyright 2020 The Cockroach Authors.
//
// Use of this software is governed by the Business Source License
// included in the file licenses/BSL.txt.
//
// As of the Change Date specified in that file, in accordance with
// the Business Source License, use of this software will be governed
// by the Apache License, Version 2.0, included in the file
// licenses/APL.txt.

package gc

import (
"context"
"fmt"
"math/rand"
"sort"
"testing"
"time"

"github.com/cockroachdb/cockroach/pkg/keys"
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/storage/engine"
"github.com/cockroachdb/cockroach/pkg/storage/engine/enginepb"
"github.com/cockroachdb/cockroach/pkg/storage/rditer"
"github.com/cockroachdb/cockroach/pkg/util/encoding"
"github.com/cockroachdb/cockroach/pkg/util/hlc"
"github.com/cockroachdb/cockroach/pkg/util/uuid"
"github.com/stretchr/testify/require"
)

// dataDistribution is an abstraction for testing that represents a stream of
// MVCCKeyValues. The stream may indicate that a value is an intent by returning
// a non-nil transaction. If an intent is returned it must have a higher
// timestamp than any other version written for the key.
type dataDistribution func() (engine.MVCCKeyValue, *roachpb.Transaction, bool)

// setupTest writes the data from this distribution into eng. All data should
// be a part of the range represented by desc.
func (ds dataDistribution) setupTest(
t testing.TB, eng engine.Engine, desc roachpb.RangeDescriptor,
) enginepb.MVCCStats {
ctx := context.TODO()
var maxTs hlc.Timestamp
var ms enginepb.MVCCStats
for {
kv, txn, ok := ds()
if !ok {
break
}
if txn == nil {
require.NoError(t, eng.Put(kv.Key, kv.Value))
} else {
// TODO(ajwerner): Decide if using MVCCPut is worth it.
ts := kv.Key.Timestamp
if txn.ReadTimestamp == (hlc.Timestamp{}) {
txn.ReadTimestamp = ts
}
if txn.WriteTimestamp == (hlc.Timestamp{}) {
txn.WriteTimestamp = ts
}
err := engine.MVCCPut(ctx, eng, &ms, kv.Key.Key, ts,
roachpb.Value{RawBytes: kv.Value}, txn)
require.NoError(t, err)
}
if !kv.Key.Timestamp.Less(maxTs) {
maxTs = kv.Key.Timestamp
}
}
require.NoError(t, eng.Flush())
snap := eng.NewSnapshot()
defer snap.Close()
ms, err := rditer.ComputeStatsForRange(&desc, snap, maxTs.WallTime)
require.NoError(t, err)
return ms
}

// newDataDistribution constructs a dataDistribution from various underlying
// distributions.
func newDataDistribution(
tsDist func() hlc.Timestamp,
keyDist func() roachpb.Key,
valueDist func() []byte,
versionsPerKey func() int,
intentFrac float64,
totalKeys int,
rng *rand.Rand,
) dataDistribution {
// TODO(ajwerner): provide a mechanism to control the rate of expired intents
// or the intent age. Such a knob would likely require decoupling intents from
// other keys.
var (
remaining = totalKeys
key roachpb.Key
seen = map[string]struct{}{}
timestamps []hlc.Timestamp
haveIntent bool
)
return func() (engine.MVCCKeyValue, *roachpb.Transaction, bool) {
if remaining == 0 {
return engine.MVCCKeyValue{}, nil, false
}
defer func() { remaining-- }()
for len(timestamps) == 0 {
versions := versionsPerKey()
if versions == 0 {
continue
}
if versions > remaining {
versions = remaining
}
timestamps = make([]hlc.Timestamp, 0, versions)
for i := 0; i < versions; i++ {
timestamps = append(timestamps, tsDist())
}
sort.Slice(timestamps, func(i, j int) bool {
return timestamps[i].Less(timestamps[j])
})
for {
key = keyDist()
sk := string(key)
if _, ok := seen[sk]; ok {
continue
}
seen[sk] = struct{}{}
break
}
haveIntent = rng.Float64() < intentFrac
}
ts := timestamps[0]
timestamps = timestamps[1:]
var txn *roachpb.Transaction
if len(timestamps) == 0 && haveIntent {
txn = &roachpb.Transaction{
Status: roachpb.PENDING,
ReadTimestamp: ts,
MaxTimestamp: ts.Next().Next(),
}
txn.ID = uuid.MakeV4()
txn.WriteTimestamp = ts
txn.Key = keyDist()
}
return engine.MVCCKeyValue{
Key: engine.MVCCKey{Key: key, Timestamp: ts},
Value: valueDist(),
}, txn, true
}
}

// distSpec abstractly represents a distribution.
type distSpec interface {
dist(maxRows int, rng *rand.Rand) dataDistribution
desc() *roachpb.RangeDescriptor
String() string
}

// uniformDistSpec is a distSpec which represents uniform distributions over its
// various dimensions.
type uniformDistSpec struct {
tsFrom, tsTo int64 // seconds
keySuffixMin, keySuffixMax int
valueLenMin, valueLenMax int
deleteFrac float64
keysPerValueMin, keysPerValueMax int
intentFrac float64
}

var _ distSpec = uniformDistSpec{}

func (ds uniformDistSpec) dist(maxRows int, rng *rand.Rand) dataDistribution {
return newDataDistribution(
uniformTimestampDistribution(ds.tsFrom*time.Second.Nanoseconds(), ds.tsTo*time.Second.Nanoseconds(), rng),
uniformTableKeyDistribution(ds.desc().StartKey.AsRawKey(), ds.keySuffixMin, ds.keySuffixMax, rng),
uniformValueDistribution(ds.valueLenMin, ds.valueLenMax, ds.deleteFrac, rng),
uniformValuesPerKey(ds.keysPerValueMin, ds.keysPerValueMax, rng),
ds.intentFrac,
maxRows,
rng,
)
}

func (ds uniformDistSpec) desc() *roachpb.RangeDescriptor {
tablePrefix := keys.MakeTablePrefix(42)
return &roachpb.RangeDescriptor{
StartKey: tablePrefix,
EndKey: roachpb.RKey(roachpb.Key(tablePrefix).PrefixEnd()),
}
}

func (ds uniformDistSpec) String() string {
return fmt.Sprintf(
"ts=[%d,%d],"+
"keySuffix=[%d,%d],"+
"valueLen=[%d,%d],"+
"keysPerValue=[%d,%d],"+
"deleteFrac=%f,intentFrac=%f",
ds.tsFrom, ds.tsTo,
ds.keySuffixMin, ds.keySuffixMax,
ds.valueLenMin, ds.valueLenMax,
ds.keysPerValueMin, ds.keysPerValueMax,
ds.deleteFrac, ds.intentFrac)
}

// uniformTimestamp returns an hlc timestamp distribution with a wall time
// uniform over [from, to] and a zero logical timestamp.
func uniformTimestampDistribution(from, to int64, rng *rand.Rand) func() hlc.Timestamp {
if from >= to {
panic(fmt.Errorf("from (%d) >= to (%d)", from, to))
}
n := int(to-from) + 1
return func() hlc.Timestamp {
return hlc.Timestamp{WallTime: from + int64(rng.Intn(n))}
}
}

// returns a uniform length random value distribution.
func uniformValueDistribution(min, max int, deleteFrac float64, rng *rand.Rand) func() []byte {
if min > max {
panic(fmt.Errorf("min (%d) > max (%d)", min, max))
}
n := (max - min) + 1
return func() []byte {
if rng.Float64() < deleteFrac {
return nil
}
value := make([]byte, min+rng.Intn(n))
if _, err := rng.Read(value); err != nil {
panic(err)
}
return value
}
}

func uniformValuesPerKey(valuesPerKeyMin, valuesPerKeyMax int, rng *rand.Rand) func() int {
if valuesPerKeyMin > valuesPerKeyMax {
panic(fmt.Errorf("min (%d) > max (%d)", valuesPerKeyMin, valuesPerKeyMax))
}
n := (valuesPerKeyMax - valuesPerKeyMin) + 1
return func() int { return valuesPerKeyMin + rng.Intn(n) }
}

func uniformTableKeyDistribution(
prefix roachpb.Key, suffixMin, suffixMax int, rng *rand.Rand,
) func() roachpb.Key {
if suffixMin > suffixMax {
panic(fmt.Errorf("suffixMin (%d) > suffixMax (%d)", suffixMin, suffixMax))
}
n := (suffixMax - suffixMin) + 1
return func() roachpb.Key {
randData := make([]byte, suffixMin+rng.Intn(n))
_, _ = rng.Read(randData)
return encoding.EncodeBytesAscending(prefix[0:len(prefix):len(prefix)], randData)
}
}

0 comments on commit 55b4936

Please sign in to comment.