Skip to content

Commit

Permalink
kvserver: use EncodedError in SnapshotResponse
Browse files Browse the repository at this point in the history
We were previously using a "Message" string to indicate details about an
error. We can do so much better now and actually encode the error. This
wasn't possible when this field was first added, but it is now, so let's
use it. As always, there's a migration concern, which means the old
field stays around & is populated as well as interpreted for one
release.

We then use this new-found freedom to improve which errors were marked
as "failed snapshot" errors. Previously, any error coming in on a
`SnapshotResponse` were considered snapshot errors and were considered
retriable. This was causing `TestLearnerSnapshotFailsRollback` to run
for 90s, as `TestCluster`'s replication changes use a [SucceedsSoon] to
retry snapshot errors - but that test actually injects an error that it
wants to fail-fast.  Now, since snapshot error marks propagate over the
wire, we can do the marking on the *sender* of the SnapshotResponse, and
we can only mark messages that correspond to an actual failure to apply
the snapshot (as opposed to an injected error, or a hard error due to a
malformed request). The test now takes around one second, for a rare 90x
speed-up.

As a drive-by, we're also removing `errMalformedSnapshot`, which became
unused when we stopped sending the raft log in raft snaps a few releases
back, and which had managed to hide from the `unused` lint.

[SucceedsSoon]: https://github.com/cockroachdb/cockroach/blob/37175f77bf374d1bcb76bc39a65149788be06134/pkg/testutils/testcluster/testcluster.go#L628-L631

Fixes #74621.

Release note: None
  • Loading branch information
tbg committed Mar 30, 2022
1 parent 350b22f commit 641c494
Show file tree
Hide file tree
Showing 7 changed files with 60 additions and 35 deletions.
2 changes: 2 additions & 0 deletions pkg/kv/kvserver/kvserverpb/BUILD.bazel
Original file line number Diff line number Diff line change
Expand Up @@ -36,6 +36,7 @@ proto_library(
"//pkg/roachpb:roachpb_proto",
"//pkg/storage/enginepb:enginepb_proto",
"//pkg/util/hlc:hlc_proto",
"@com_github_cockroachdb_errors//errorspb:errorspb_proto",
"@com_github_gogo_protobuf//gogoproto:gogo_proto",
"@com_google_protobuf//:timestamp_proto",
"@io_etcd_go_etcd_raft_v3//raftpb:raftpb_proto",
Expand All @@ -56,6 +57,7 @@ go_proto_library(
"//pkg/storage/enginepb",
"//pkg/util/hlc",
"//pkg/util/uuid", # keep
"@com_github_cockroachdb_errors//errorspb",
"@com_github_gogo_protobuf//gogoproto",
"@io_etcd_go_etcd_raft_v3//raftpb",
],
Expand Down
11 changes: 11 additions & 0 deletions pkg/kv/kvserver/kvserverpb/raft.proto
Original file line number Diff line number Diff line change
Expand Up @@ -12,6 +12,7 @@ syntax = "proto3";
package cockroach.kv.kvserver.kvserverpb;
option go_package = "kvserverpb";

import "errorspb/errors.proto";
import "roachpb/errors.proto";
import "roachpb/metadata.proto";
import "kv/kvserver/liveness/livenesspb/liveness.proto";
Expand Down Expand Up @@ -210,8 +211,18 @@ message SnapshotResponse {
reserved 4;
}
Status status = 1;
// Message is a message explaining an ERROR return value. It is not set for any
// other status.
//
// DEPRECATED: this field can be removed in 23.1. As of 22.2, the encoded_error
// field is always used instead. (22.2 itself needs to populate both due to
// needing to be wire-compatible with 22.1).
string message = 2;
reserved 3;
// encoded_error encodes the error when the status is ERROR.
//
// MIGRATION: only guaranteed to be set when the message field is no longer there.
errorspb.EncodedError encoded_error = 4 [(gogoproto.nullable) = false];
}

// ConfChangeContext is encoded in the raftpb.ConfChange.Context field.
Expand Down
4 changes: 4 additions & 0 deletions pkg/kv/kvserver/markers.go
Original file line number Diff line number Diff line change
Expand Up @@ -15,6 +15,10 @@ import (
"github.com/cockroachdb/errors"
)

// errMarkSnapshotError is used as an error mark for errors that get returned
// to the initiator of a snapshot. This generally classifies errors as transient,
// i.e. communicates an intention for the caller to retry.
//
// NB: don't change the string here; this will cause cross-version issues
// since this singleton is used as a marker.
var errMarkSnapshotError = errors.New("snapshot failed")
Expand Down
7 changes: 5 additions & 2 deletions pkg/kv/kvserver/raft_transport.go
Original file line number Diff line number Diff line change
Expand Up @@ -424,9 +424,12 @@ func (t *RaftTransport) RaftSnapshot(stream MultiRaft_RaftSnapshotServer) error
return err
}
if req.Header == nil {
err := errors.New("client error: no header in first snapshot request message")
return stream.Send(&kvserverpb.SnapshotResponse{
Status: kvserverpb.SnapshotResponse_ERROR,
Message: "client error: no header in first snapshot request message"})
Status: kvserverpb.SnapshotResponse_ERROR,
Message: err.Error(),
EncodedError: errors.EncodeError(ctx, err),
})
}
rmr := req.Header.RaftMessageRequest
handler, ok := t.getHandler(rmr.ToReplica.StoreID)
Expand Down
16 changes: 1 addition & 15 deletions pkg/kv/kvserver/replica_command.go
Original file line number Diff line number Diff line change
Expand Up @@ -2582,7 +2582,7 @@ func (r *Replica) sendSnapshot(
sent := func() {
r.store.metrics.RangeSnapshotsGenerated.Inc(1)
}
err = contextutil.RunWithTimeout(
return contextutil.RunWithTimeout(
ctx, "send-snapshot", sendSnapshotTimeout, func(ctx context.Context) error {
return r.store.cfg.Transport.SendSnapshot(
ctx,
Expand All @@ -2594,20 +2594,6 @@ func (r *Replica) sendSnapshot(
r.store.metrics.RangeSnapshotSentBytes,
)
})
if err != nil {
if errors.Is(err, errMalformedSnapshot) {
tag := fmt.Sprintf("r%d_%s", r.RangeID, snap.SnapUUID.Short())
if dir, err := r.store.checkpoint(ctx, tag); err != nil {
log.Warningf(ctx, "unable to create checkpoint %s: %+v", dir, err)
} else {
log.Warningf(ctx, "created checkpoint %s", dir)
}

log.Fatal(ctx, "malformed snapshot generated")
}
return errors.Mark(err, errMarkSnapshotError)
}
return nil
}

// replicasCollocated is used in AdminMerge to ensure that the ranges are
Expand Down
8 changes: 4 additions & 4 deletions pkg/kv/kvserver/replica_learner_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -316,11 +316,11 @@ func TestLearnerSnapshotFailsRollback(t *testing.T) {
skip.UnderRace(t)

runTest := func(t *testing.T, replicaType roachpb.ReplicaType) {
var rejectSnapshots int64
var rejectSnapshotErr atomic.Value // error
knobs, ltk := makeReplicationTestKnobs()
ltk.storeKnobs.ReceiveSnapshot = func(h *kvserverpb.SnapshotRequest_Header) error {
if atomic.LoadInt64(&rejectSnapshots) > 0 {
return errors.New(`nope`)
if err := rejectSnapshotErr.Load().(error); err != nil {
return err
}
return nil
}
Expand All @@ -332,7 +332,7 @@ func TestLearnerSnapshotFailsRollback(t *testing.T) {
defer tc.Stopper().Stop(ctx)

scratchStartKey := tc.ScratchRange(t)
atomic.StoreInt64(&rejectSnapshots, 1)
rejectSnapshotErr.Store(errors.New("boom"))
var err error
switch replicaType {
case roachpb.LEARNER:
Expand Down
47 changes: 33 additions & 14 deletions pkg/kv/kvserver/store_snapshot.go
Original file line number Diff line number Diff line change
Expand Up @@ -330,10 +330,6 @@ func (kvSS *kvBatchSnapshotStrategy) Receive(
}
}

// errMalformedSnapshot indicates that the snapshot in question is malformed,
// for e.g. missing raft log entries.
var errMalformedSnapshot = errors.New("malformed snapshot generated")

// Send implements the snapshotStrategy interface.
func (kvSS *kvBatchSnapshotStrategy) Send(
ctx context.Context,
Expand Down Expand Up @@ -563,7 +559,10 @@ func (s *Store) checkSnapshotOverlapLocked(
// NB: this check seems redundant since placeholders are also represented in
// replicasByKey (and thus returned in getOverlappingKeyRangeLocked).
if exRng, ok := s.mu.replicaPlaceholders[desc.RangeID]; ok {
return errors.Errorf("%s: canAcceptSnapshotLocked: cannot add placeholder, have an existing placeholder %s %v", s, exRng, snapHeader.RaftMessageRequest.FromReplica)
return errors.Mark(errors.Errorf(
"%s: canAcceptSnapshotLocked: cannot add placeholder, have an existing placeholder %s %v",
s, exRng, snapHeader.RaftMessageRequest.FromReplica),
errMarkSnapshotError)
}

// TODO(benesch): consider discovering and GC'ing *all* overlapping ranges,
Expand Down Expand Up @@ -608,7 +607,10 @@ func (s *Store) checkSnapshotOverlapLocked(
msg += "; initiated GC:"
s.replicaGCQueue.AddAsync(ctx, exReplica, gcPriority)
}
return errors.Errorf("%s %v (incoming %v)", msg, exReplica, snapHeader.State.Desc.RSpan()) // exReplica can be nil
return errors.Mark(
errors.Errorf("%s %v (incoming %v)", msg, exReplica, snapHeader.State.Desc.RSpan()), // exReplica can be nil
errMarkSnapshotError,
)
}
return nil
}
Expand Down Expand Up @@ -640,6 +642,8 @@ func (s *Store) receiveSnapshot(

if fn := s.cfg.TestingKnobs.ReceiveSnapshot; fn != nil {
if err := fn(header); err != nil {
// NB: we intentionally don't mark this error as errMarkSnapshotError so
// that we don't end up retrying injected errors in tests.
return sendSnapshotError(stream, err)
}
}
Expand Down Expand Up @@ -680,7 +684,7 @@ func (s *Store) receiveSnapshot(
return nil
}); pErr != nil {
log.Infof(ctx, "cannot accept snapshot: %s", pErr)
return pErr.GoError()
return sendSnapshotError(stream, pErr.GoError())
}

defer func() {
Expand Down Expand Up @@ -736,16 +740,22 @@ func (s *Store) receiveSnapshot(
// already received the entire snapshot here, so there's no point in
// abandoning application half-way through if the caller goes away.
applyCtx := s.AnnotateCtx(context.Background())
if err := s.processRaftSnapshotRequest(applyCtx, header, inSnap); err != nil {
return sendSnapshotError(stream, errors.Wrap(err.GoError(), "failed to apply snapshot"))
if pErr := s.processRaftSnapshotRequest(applyCtx, header, inSnap); pErr != nil {
err := pErr.GoError()
// We mark this error as a snapshot error which will be interpreted by the
// sender as this being a retriable error, see isSnapshotError().
err = errors.Mark(err, errMarkSnapshotError)
err = errors.Wrap(err, "failed to apply snapshot")
return sendSnapshotError(stream, err)
}
return stream.Send(&kvserverpb.SnapshotResponse{Status: kvserverpb.SnapshotResponse_APPLIED})
}

func sendSnapshotError(stream incomingSnapshotStream, err error) error {
return stream.Send(&kvserverpb.SnapshotResponse{
Status: kvserverpb.SnapshotResponse_ERROR,
Message: err.Error(),
Status: kvserverpb.SnapshotResponse_ERROR,
Message: err.Error(),
EncodedError: errors.EncodeError(context.Background(), err),
})
}

Expand Down Expand Up @@ -1135,9 +1145,18 @@ func sendSnapshot(
}
switch resp.Status {
case kvserverpb.SnapshotResponse_ERROR:
storePool.throttle(throttleFailed, resp.Message, to.StoreID)
return errors.Errorf("%s: remote couldn't accept %s with error: %s",
to, snap, resp.Message)
var err error
if resp.EncodedError.Error != nil {
// NB: errMarkSnapshotError must be set on the other end, if it is
// set. We're not going to add it here.
err = errors.DecodeError(ctx, resp.EncodedError)
} else {
// Deprecated path.
err = errors.Errorf("%s", resp.Message)
err = errors.Mark(err, errMarkSnapshotError)
}
storePool.throttle(throttleFailed, err.Error(), to.StoreID)
return errors.Wrapf(err, "%s: remote couldn't accept %s", to, snap)
case kvserverpb.SnapshotResponse_ACCEPTED:
// This is the response we're expecting. Continue with snapshot sending.
default:
Expand Down

0 comments on commit 641c494

Please sign in to comment.