@@ -87,24 +87,23 @@ func (rc ReadConsistencyType) SupportsBatch(ba BatchRequest) error {
8787}
8888
8989const (
90- isAdmin = 1 << iota // admin cmds don't go through raft, but run on lease holder
91- isRead // read-only cmds don't go through raft, but may run on lease holder
92- isWrite // write cmds go through raft and must be proposed on lease holder
93- isTxn // txn commands may be part of a transaction
94- isTxnWrite // txn write cmds start heartbeat and are marked for intent resolution
95- isRange // range commands may span multiple keys
96- isReverse // reverse commands traverse ranges in descending direction
97- isAlone // requests which must be alone in a batch
98- isPrefix // requests which should be grouped with the next request in a batch
99- isUnsplittable // range command that must not be split during sending
100- // Requests for acquiring a lease skip the (proposal-time) check that the
101- // proposing replica has a valid lease.
102- skipLeaseCheck
103- consultsTSCache // mutating commands which write data at a timestamp
104- updatesReadTSCache // commands which update the read timestamp cache
105- updatesWriteTSCache // commands which update the write timestamp cache
106- updatesTSCacheOnError // commands which make read data available on errors
107- needsRefresh // commands which require refreshes to avoid serializable retries
90+ isAdmin = 1 << iota // admin cmds don't go through raft, but run on lease holder
91+ isRead // read-only cmds don't go through raft, but may run on lease holder
92+ isWrite // write cmds go through raft and must be proposed on lease holder
93+ isTxn // txn commands may be part of a transaction
94+ isTxnWrite // txn write cmds start heartbeat and are marked for intent resolution
95+ isRange // range commands may span multiple keys
96+ isReverse // reverse commands traverse ranges in descending direction
97+ isAlone // requests which must be alone in a batch
98+ isPrefix // requests which should be grouped with the next request in a batch
99+ isUnsplittable // range command that must not be split during sending
100+ skipLeaseCheck // commands which skip the check that the evaluting replica has a valid lease
101+ consultsTSCache // mutating commands which write data at a timestamp
102+ updatesReadTSCache // commands which update the read timestamp cache
103+ updatesWriteTSCache // commands which update the write timestamp cache
104+ updatesTSCacheOnErr // commands which make read data available on errors
105+ needsRefresh // commands which require refreshes to avoid serializable retries
106+ canBackpressure // commands which deserve backpressure when a Range grows too large
108107)
109108
110109// IsReadOnly returns true iff the request is read-only.
@@ -157,7 +156,7 @@ func UpdatesWriteTimestampCache(args Request) bool {
157156// update the timestamp cache even on error, as in some cases the data
158157// which was read is returned (e.g. ConditionalPut ConditionFailedError).
159158func UpdatesTimestampCacheOnError (args Request ) bool {
160- return (args .flags () & updatesTSCacheOnError ) != 0
159+ return (args .flags () & updatesTSCacheOnErr ) != 0
161160}
162161
163162// NeedsRefresh returns whether the command must be refreshed in
@@ -166,6 +165,12 @@ func NeedsRefresh(args Request) bool {
166165 return (args .flags () & needsRefresh ) != 0
167166}
168167
168+ // CanBackpressure returns whether the command can be backpressured
169+ // when waiting for a Range to split after it has grown too large.
170+ func CanBackpressure (args Request ) bool {
171+ return (args .flags () & canBackpressure ) != 0
172+ }
173+
169174// Request is an interface for RPC requests.
170175type Request interface {
171176 protoutil.Message
@@ -948,7 +953,9 @@ func NewReverseScan(key, endKey Key) Request {
948953}
949954
950955func (* GetRequest ) flags () int { return isRead | isTxn | updatesReadTSCache | needsRefresh }
951- func (* PutRequest ) flags () int { return isWrite | isTxn | isTxnWrite | consultsTSCache }
956+ func (* PutRequest ) flags () int {
957+ return isWrite | isTxn | isTxnWrite | consultsTSCache | canBackpressure
958+ }
952959
953960// ConditionalPut effectively reads and may not write, so must update
954961// the timestamp cache. Note that on ConditionFailedErrors
@@ -957,7 +964,7 @@ func (*PutRequest) flags() int { return isWrite | isTxn | isTxnWrite | consultsT
957964// errors, they return an error immediately instead of continuing a
958965// serializable transaction to be retried at end transaction.
959966func (* ConditionalPutRequest ) flags () int {
960- return isRead | isWrite | isTxn | isTxnWrite | updatesReadTSCache | updatesTSCacheOnError | consultsTSCache
967+ return isRead | isWrite | isTxn | isTxnWrite | consultsTSCache | updatesReadTSCache | updatesTSCacheOnErr | canBackpressure
961968}
962969
963970// InitPut, like ConditionalPut, effectively reads and may not write.
@@ -967,7 +974,7 @@ func (*ConditionalPutRequest) flags() int {
967974// immediately instead of continuing a serializable transaction to be
968975// retried at end transaction.
969976func (* InitPutRequest ) flags () int {
970- return isRead | isWrite | isTxn | isTxnWrite | updatesReadTSCache | updatesTSCacheOnError | consultsTSCache
977+ return isRead | isWrite | isTxn | isTxnWrite | consultsTSCache | updatesReadTSCache | updatesTSCacheOnErr | canBackpressure
971978}
972979
973980// Increment reads the existing value, but always leaves an intent so
@@ -976,10 +983,12 @@ func (*InitPutRequest) flags() int {
976983// error immediately instead of continuing a serializable transaction
977984// to be retried at end transaction.
978985func (* IncrementRequest ) flags () int {
979- return isRead | isWrite | isTxn | isTxnWrite | consultsTSCache
986+ return isRead | isWrite | isTxn | isTxnWrite | consultsTSCache | canBackpressure
980987}
981988
982- func (* DeleteRequest ) flags () int { return isWrite | isTxn | isTxnWrite | consultsTSCache }
989+ func (* DeleteRequest ) flags () int {
990+ return isWrite | isTxn | isTxnWrite | consultsTSCache | canBackpressure
991+ }
983992func (drr * DeleteRangeRequest ) flags () int {
984993 // DeleteRangeRequest has different properties if the "inline" flag is set.
985994 // This flag indicates that the request is deleting inline MVCC values,
@@ -1002,7 +1011,7 @@ func (drr *DeleteRangeRequest) flags() int {
10021011 // intents or tombstones for keys which don't yet exist. By updating
10031012 // the write timestamp cache, it forces subsequent writes to get a
10041013 // write-too-old error and avoids the phantom delete anomaly.
1005- return isWrite | isTxn | isTxnWrite | isRange | updatesWriteTSCache | needsRefresh | consultsTSCache
1014+ return isWrite | isTxn | isTxnWrite | isRange | consultsTSCache | updatesWriteTSCache | needsRefresh | canBackpressure
10061015}
10071016
10081017// Note that ClearRange commands cannot be part of a transaction as
@@ -1041,7 +1050,7 @@ func (*QueryIntentRequest) flags() int { return isRead | isPrefix | updat
10411050func (* ResolveIntentRequest ) flags () int { return isWrite }
10421051func (* ResolveIntentRangeRequest ) flags () int { return isWrite | isRange }
10431052func (* TruncateLogRequest ) flags () int { return isWrite }
1044- func (* MergeRequest ) flags () int { return isWrite }
1053+ func (* MergeRequest ) flags () int { return isWrite | canBackpressure }
10451054func (* RequestLeaseRequest ) flags () int { return isWrite | isAlone | skipLeaseCheck }
10461055
10471056// LeaseInfoRequest is usually executed in an INCONSISTENT batch, which has the
@@ -1068,8 +1077,10 @@ func (*CheckConsistencyRequest) flags() int { return isAdmin | isRange }
10681077func (* WriteBatchRequest ) flags () int { return isWrite | isRange }
10691078func (* ExportRequest ) flags () int { return isRead | isRange | updatesReadTSCache }
10701079func (* ImportRequest ) flags () int { return isAdmin | isAlone }
1071- func (* AdminScatterRequest ) flags () int { return isAdmin | isAlone | isRange }
1072- func (* AddSSTableRequest ) flags () int { return isWrite | isAlone | isRange | isUnsplittable }
1080+ func (* AdminScatterRequest ) flags () int { return isAdmin | isRange | isAlone }
1081+ func (* AddSSTableRequest ) flags () int {
1082+ return isWrite | isRange | isAlone | isUnsplittable | canBackpressure
1083+ }
10731084
10741085// RefreshRequest and RefreshRangeRequest both determine which timestamp cache
10751086// they update based on their Write parameter.
0 commit comments