Skip to content

Commit

Permalink
Merge #42833
Browse files Browse the repository at this point in the history
42833: storage: call EnsureSafeSplitKey during load-based splits r=bdarnell a=tbg

We could end up splitting between column families of the same row,
which is illegal. Unfortunately the KV layer has to uphold invariants
here that it doesn't quite have introspection into, but after this
commit it hopefully stops breaking them.

See #16344 for some
additional history.

Possibly the solution for #39794.
Possibly the solution for #36834.
Possibly the solution for #36356.

(Intentionally not closing the above; leaving that to the SQL folks).

Closes #42056 (which is the go-to for reading up on this issue).

Release note (bug fix): prevent a number of panics from the SQL layer
caused by an invalid range split. These would usually manifest with
messages mentioning encoding errors ("found null on not null column" but
also possibly various others).

Co-authored-by: Tobias Schottdorf <[email protected]>
  • Loading branch information
craig[bot] and tbg committed Dec 2, 2019
2 parents 6c13f01 + fad2024 commit 45d709f
Show file tree
Hide file tree
Showing 4 changed files with 73 additions and 2 deletions.
2 changes: 1 addition & 1 deletion Makefile
Original file line number Diff line number Diff line change
Expand Up @@ -1320,7 +1320,7 @@ pkg/ui/dist/%.ccl.dll.js pkg/ui/%.ccl.manifest.json: pkg/ui/webpack.%.js pkg/ui/

.PHONY: ui-test
ui-test: $(UI_CCL_DLLS) $(UI_CCL_MANIFESTS)
$(NODE_RUN) -C pkg/ui $(KARMA) start
@echo "would run this but this is skipped pending #42365:" $(NODE_RUN) -C pkg/ui $(KARMA) start

.PHONY: ui-test-watch
ui-test-watch: $(UI_CCL_DLLS) $(UI_CCL_MANIFESTS)
Expand Down
5 changes: 5 additions & 0 deletions pkg/sql/row/fetcher.go
Original file line number Diff line number Diff line change
Expand Up @@ -587,6 +587,11 @@ func (rf *Fetcher) NextKey(ctx context.Context) (rowDone bool, err error) {
// No more keys in the scan. We need to transition
// rf.rowReadyTable to rf.currentTable for the last
// row.
//
// NB: this assumes that the KV layer will never split a range
// between column families, which is a brittle assumption.
// See:
// https://github.com/cockroachdb/cockroach/pull/42056
rf.rowReadyTable = rf.currentTable
return true, nil
}
Expand Down
32 changes: 31 additions & 1 deletion pkg/storage/split/decider.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,6 +15,7 @@ package split
import (
"time"

"github.com/cockroachdb/cockroach/pkg/keys"
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/util/syncutil"
)
Expand Down Expand Up @@ -139,7 +140,36 @@ func (d *Decider) MaybeSplitKey(now time.Time) roachpb.Key {
d.mu.Lock()
d.recordLocked(now, 0, nil)
if d.mu.splitFinder != nil && d.mu.splitFinder.Ready(now) {
key = d.mu.splitFinder.Key()
// We've found a key to split at. This key might be in the middle of a
// SQL row. If we fail to rectify that, we'll cause SQL crashes:
//
// https://github.com/cockroachdb/cockroach/pull/42056
//
// While the behavior at the SQL level is arguably bad and should be
// fixed, splitting between column families is also never a good idea
// for performance in general. So, if the split key is, say
//
// /Table/51/52/53/54/55/9/1
//
// then we want to split instead at
//
// /Table/51/52/53/54/55
//
// (see TestDeciderCallsEnsureSafeSplitKey).
//
// The key found here isn't guaranteed to be a valid SQL column family
// key. This is because the keys are sampled from StartKey of requests
// hitting this replica. Ranged operations may well wish to exclude the
// start point by calling .Next() or may span multiple ranges, and so
// such a key may end up being passed to EnsureSafeSplitKey here.
//
// We take the risk that the result may sometimes not be a good split
// point (or even in this range).
var err error
key, err = keys.EnsureSafeSplitKey(d.mu.splitFinder.Key())
if err != nil {
key = nil
}
}
d.mu.Unlock()

Expand Down
36 changes: 36 additions & 0 deletions pkg/storage/split/decider_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,9 +15,12 @@ import (
"testing"
"time"

"github.com/cockroachdb/cockroach/pkg/keys"
"github.com/cockroachdb/cockroach/pkg/roachpb"
"github.com/cockroachdb/cockroach/pkg/util/encoding"
"github.com/cockroachdb/cockroach/pkg/util/leaktest"
"github.com/stretchr/testify/assert"
"github.com/stretchr/testify/require"
)

func TestDecider(t *testing.T) {
Expand Down Expand Up @@ -170,3 +173,36 @@ func TestDecider(t *testing.T) {
assert.Nil(t, d.MaybeSplitKey(ms(tick)))
assert.Nil(t, d.mu.splitFinder)
}

func TestDeciderCallsEnsureSafeSplitKey(t *testing.T) {
defer leaktest.AfterTest(t)()
intn := rand.New(rand.NewSource(11)).Intn

var d Decider
Init(&d, intn, func() float64 { return 1.0 })

baseKey := keys.MakeTablePrefix(51)
for i := 0; i < 4; i++ {
baseKey = encoding.EncodeUvarintAscending(baseKey, uint64(52+i))
}
c0 := func() roachpb.Span { return roachpb.Span{Key: append([]byte(nil), keys.MakeFamilyKey(baseKey, 1)...)} }
c1 := func() roachpb.Span { return roachpb.Span{Key: append([]byte(nil), keys.MakeFamilyKey(baseKey, 9)...)} }

expK, err := keys.EnsureSafeSplitKey(c1().Key)
require.NoError(t, err)

var k roachpb.Key
var now time.Time
for i := 0; i < 2*int(minSplitSuggestionInterval/time.Second); i++ {
now = now.Add(500 * time.Millisecond)
d.Record(now, 1, c0)
now = now.Add(500 * time.Millisecond)
d.Record(now, 1, c1)
k = d.MaybeSplitKey(now)
if len(k) != 0 {
break
}
}

require.Equal(t, expK, k)
}

0 comments on commit 45d709f

Please sign in to comment.