HBASE-26412 Handle sink failure in RegionReplicationSink (#3815)
[hbase.git] / hbase-server / src / main / java / org / apache / hadoop / hbase / regionserver / Region.java
blob2b8502b4fd0ae5c260adc0fff993f2abbbb61bd7
1 /*
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase.regionserver;
20 import java.io.IOException;
21 import java.util.Collection;
22 import java.util.List;
23 import java.util.Map;
25 import org.apache.hadoop.conf.Configuration;
26 import org.apache.hadoop.hbase.Cell;
27 import org.apache.hadoop.hbase.CellComparator;
28 import org.apache.hadoop.hbase.CompareOperator;
29 import org.apache.hadoop.hbase.HBaseInterfaceAudience;
30 import org.apache.hadoop.hbase.client.Append;
31 import org.apache.hadoop.hbase.client.CheckAndMutate;
32 import org.apache.hadoop.hbase.client.CheckAndMutateResult;
33 import org.apache.hadoop.hbase.client.CompactionState;
34 import org.apache.hadoop.hbase.client.Delete;
35 import org.apache.hadoop.hbase.client.Get;
36 import org.apache.hadoop.hbase.client.Increment;
37 import org.apache.hadoop.hbase.client.Mutation;
38 import org.apache.hadoop.hbase.client.Put;
39 import org.apache.hadoop.hbase.client.RegionInfo;
40 import org.apache.hadoop.hbase.client.Result;
41 import org.apache.hadoop.hbase.client.RowMutations;
42 import org.apache.hadoop.hbase.client.Scan;
43 import org.apache.hadoop.hbase.client.TableDescriptor;
44 import org.apache.hadoop.hbase.conf.ConfigurationObserver;
45 import org.apache.hadoop.hbase.coprocessor.RegionCoprocessor;
46 import org.apache.hadoop.hbase.filter.ByteArrayComparable;
47 import org.apache.hadoop.hbase.filter.Filter;
48 import org.apache.hadoop.hbase.io.TimeRange;
49 import org.apache.hadoop.hbase.regionserver.compactions.CompactionLifeCycleTracker;
50 import org.apache.yetus.audience.InterfaceAudience;
51 import org.apache.yetus.audience.InterfaceStability;
53 /**
54 * Region is a subset of HRegion with operations required for the {@link RegionCoprocessor
55 * Coprocessors}. The operations include ability to do mutations, requesting compaction, getting
56 * different counters/sizes, locking rows and getting access to {@linkplain Store}s.
58 @InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.COPROC)
59 @InterfaceStability.Evolving
60 public interface Region extends ConfigurationObserver {
62 ///////////////////////////////////////////////////////////////////////////
63 // Region state
65 /** @return region information for this region */
66 RegionInfo getRegionInfo();
68 /** @return table descriptor for this region */
69 TableDescriptor getTableDescriptor();
71 /** @return true if region is available (not closed and not closing) */
72 boolean isAvailable();
74 /** @return true if region is closed */
75 boolean isClosed();
77 /** @return True if closing process has started */
78 boolean isClosing();
80 /** @return True if region is read only */
81 boolean isReadOnly();
83 /** @return true if region is splittable */
84 boolean isSplittable();
86 /**
87 * @return true if region is mergeable
89 boolean isMergeable();
91 /**
92 * Return the list of Stores managed by this region
93 * <p>Use with caution. Exposed for use of fixup utilities.
94 * @return a list of the Stores managed by this region
96 List<? extends Store> getStores();
98 /**
99 * Return the Store for the given family
100 * <p>Use with caution. Exposed for use of fixup utilities.
101 * @return the Store for the given family
103 Store getStore(byte[] family);
105 /** @return list of store file names for the given families */
106 List<String> getStoreFileList(byte[][] columns);
109 * Check the region's underlying store files, open the files that have not
110 * been opened yet, and remove the store file readers for store files no
111 * longer available.
112 * @throws IOException
114 boolean refreshStoreFiles() throws IOException;
116 /** @return the max sequence id of flushed data on this region; no edit in memory will have
117 * a sequence id that is less that what is returned here.
119 long getMaxFlushedSeqId();
122 * This can be used to determine the last time all files of this region were major compacted.
123 * @param majorCompactionOnly Only consider HFile that are the result of major compaction
124 * @return the timestamp of the oldest HFile for all stores of this region
126 long getOldestHfileTs(boolean majorCompactionOnly) throws IOException;
129 * @return map of column family names to max sequence id that was read from storage when this
130 * region was opened
132 public Map<byte[], Long> getMaxStoreSeqId();
135 * @return The earliest time a store in the region was flushed. All
136 * other stores in the region would have been flushed either at, or
137 * after this time.
139 long getEarliestFlushTimeForAllStores();
141 ///////////////////////////////////////////////////////////////////////////
142 // Metrics
144 /** @return read requests count for this region */
145 long getReadRequestsCount();
147 /** @return coprocessor requests count for this region */
148 long getCpRequestsCount();
150 /** @return filtered read requests count for this region */
151 long getFilteredReadRequestsCount();
153 /** @return write request count for this region */
154 long getWriteRequestsCount();
157 * @return memstore size for this region, in bytes. It just accounts data size of cells added to
158 * the memstores of this Region. Means size in bytes for key, value and tags within Cells.
159 * It wont consider any java heap overhead for the cell objects or any other.
161 long getMemStoreDataSize();
164 * @return memstore heap size for this region, in bytes. It accounts data size of cells
165 * added to the memstores of this Region, as well as java heap overhead for the cell
166 * objects or any other.
168 long getMemStoreHeapSize();
171 * @return memstore off-heap size for this region, in bytes. It accounts data size of cells
172 * added to the memstores of this Region, as well as overhead for the cell
173 * objects or any other that is allocated off-heap.
175 long getMemStoreOffHeapSize();
177 /** @return the number of mutations processed bypassing the WAL */
178 long getNumMutationsWithoutWAL();
180 /** @return the size of data processed bypassing the WAL, in bytes */
181 long getDataInMemoryWithoutWAL();
183 /** @return the number of blocked requests */
184 long getBlockedRequestsCount();
186 /** @return the number of checkAndMutate guards that passed */
187 long getCheckAndMutateChecksPassed();
189 /** @return the number of failed checkAndMutate guards */
190 long getCheckAndMutateChecksFailed();
192 ///////////////////////////////////////////////////////////////////////////
193 // Locking
195 // Region read locks
198 * Operation enum is used in {@link Region#startRegionOperation} and elsewhere to provide
199 * context for various checks.
201 enum Operation {
202 ANY, GET, PUT, DELETE, SCAN, APPEND, INCREMENT, SPLIT_REGION, MERGE_REGION, BATCH_MUTATE,
203 REPLAY_BATCH_MUTATE, COMPACT_REGION, REPLAY_EVENT, SNAPSHOT, COMPACT_SWITCH,
204 CHECK_AND_MUTATE
208 * This method needs to be called before any public call that reads or
209 * modifies data.
210 * Acquires a read lock and checks if the region is closing or closed.
211 * <p>{@link #closeRegionOperation} MUST then always be called after
212 * the operation has completed, whether it succeeded or failed.
213 * @throws IOException
215 // TODO Exposing this and closeRegionOperation() as we have getRowLock() exposed.
216 // Remove if we get rid of exposing getRowLock().
217 void startRegionOperation() throws IOException;
220 * This method needs to be called before any public call that reads or
221 * modifies data.
222 * Acquires a read lock and checks if the region is closing or closed.
223 * <p>{@link #closeRegionOperation} MUST then always be called after
224 * the operation has completed, whether it succeeded or failed.
225 * @param op The operation is about to be taken on the region
226 * @throws IOException
228 void startRegionOperation(Operation op) throws IOException;
231 * Closes the region operation lock.
232 * @throws IOException
234 void closeRegionOperation() throws IOException;
237 * Closes the region operation lock. This needs to be called in the finally block corresponding
238 * to the try block of {@link #startRegionOperation(Operation)}
239 * @throws IOException
241 void closeRegionOperation(Operation op) throws IOException;
243 // Row write locks
246 * Row lock held by a given thread.
247 * One thread may acquire multiple locks on the same row simultaneously.
248 * The locks must be released by calling release() from the same thread.
250 public interface RowLock {
252 * Release the given lock. If there are no remaining locks held by the current thread
253 * then unlock the row and allow other threads to acquire the lock.
254 * @throws IllegalArgumentException if called by a different thread than the lock owning
255 * thread
257 void release();
261 * Get a row lock for the specified row. All locks are reentrant.
263 * Before calling this function make sure that a region operation has already been
264 * started (the calling thread has already acquired the region-close-guard lock).
265 * <p>
266 * The obtained locks should be released after use by {@link RowLock#release()}
267 * <p>
268 * NOTE: the boolean passed here has changed. It used to be a boolean that
269 * stated whether or not to wait on the lock. Now it is whether it an exclusive
270 * lock is requested.
272 * @param row The row actions will be performed against
273 * @param readLock is the lock reader or writer. True indicates that a non-exclusive
274 * lock is requested
275 * @see #startRegionOperation()
276 * @see #startRegionOperation(Operation)
278 RowLock getRowLock(byte[] row, boolean readLock) throws IOException;
280 ///////////////////////////////////////////////////////////////////////////
281 // Region operations
284 * Perform one or more append operations on a row.
285 * @param append
286 * @return result of the operation
287 * @throws IOException
289 Result append(Append append) throws IOException;
292 * Perform a batch of mutations.
293 * <p>
294 * Please do not operate on a same column of a single row in a batch, we will not consider the
295 * previous operation in the same batch when performing the operations in the batch.
297 * @param mutations the list of mutations
298 * @return an array of OperationStatus which internally contains the
299 * OperationStatusCode and the exceptionMessage if any.
300 * @throws IOException
302 OperationStatus[] batchMutate(Mutation[] mutations)
303 throws IOException;
306 * Atomically checks if a row/family/qualifier value matches the expected value and if it does,
307 * it performs the mutation. If the passed value is null, the lack of column value
308 * (ie: non-existence) is used. See checkAndRowMutate to do many checkAndPuts at a time on a
309 * single row.
310 * @param row to check
311 * @param family column family to check
312 * @param qualifier column qualifier to check
313 * @param op the comparison operator
314 * @param comparator the expected value
315 * @param mutation data to put if check succeeds
316 * @return true if mutation was applied, false otherwise
318 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
319 * {@link #checkAndMutate(CheckAndMutate)} instead.
321 @Deprecated
322 default boolean checkAndMutate(byte [] row, byte [] family, byte [] qualifier, CompareOperator op,
323 ByteArrayComparable comparator, Mutation mutation) throws IOException {
324 return checkAndMutate(row, family, qualifier, op, comparator, TimeRange.allTime(), mutation);
328 * Atomically checks if a row/family/qualifier value matches the expected value and if it does,
329 * it performs the mutation. If the passed value is null, the lack of column value
330 * (ie: non-existence) is used. See checkAndRowMutate to do many checkAndPuts at a time on a
331 * single row.
332 * @param row to check
333 * @param family column family to check
334 * @param qualifier column qualifier to check
335 * @param op the comparison operator
336 * @param comparator the expected value
337 * @param mutation data to put if check succeeds
338 * @param timeRange time range to check
339 * @return true if mutation was applied, false otherwise
341 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
342 * {@link #checkAndMutate(CheckAndMutate)} instead.
344 @Deprecated
345 boolean checkAndMutate(byte [] row, byte [] family, byte [] qualifier, CompareOperator op,
346 ByteArrayComparable comparator, TimeRange timeRange, Mutation mutation) throws IOException;
349 * Atomically checks if a row matches the filter and if it does, it performs the mutation. See
350 * checkAndRowMutate to do many checkAndPuts at a time on a single row.
351 * @param row to check
352 * @param filter the filter
353 * @param mutation data to put if check succeeds
354 * @return true if mutation was applied, false otherwise
356 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
357 * {@link #checkAndMutate(CheckAndMutate)} instead.
359 @Deprecated
360 default boolean checkAndMutate(byte [] row, Filter filter, Mutation mutation)
361 throws IOException {
362 return checkAndMutate(row, filter, TimeRange.allTime(), mutation);
366 * Atomically checks if a row value matches the filter and if it does, it performs the mutation.
367 * See checkAndRowMutate to do many checkAndPuts at a time on a single row.
368 * @param row to check
369 * @param filter the filter
370 * @param mutation data to put if check succeeds
371 * @param timeRange time range to check
372 * @return true if mutation was applied, false otherwise
374 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
375 * {@link #checkAndMutate(CheckAndMutate)} instead.
377 @Deprecated
378 boolean checkAndMutate(byte [] row, Filter filter, TimeRange timeRange, Mutation mutation)
379 throws IOException;
382 * Atomically checks if a row/family/qualifier value matches the expected values and if it does,
383 * it performs the row mutations. If the passed value is null, the lack of column value
384 * (ie: non-existence) is used. Use to do many mutations on a single row. Use checkAndMutate
385 * to do one checkAndMutate at a time.
386 * @param row to check
387 * @param family column family to check
388 * @param qualifier column qualifier to check
389 * @param op the comparison operator
390 * @param comparator the expected value
391 * @param mutations data to put if check succeeds
392 * @return true if mutations were applied, false otherwise
394 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
395 * {@link #checkAndMutate(CheckAndMutate)} instead.
397 @Deprecated
398 default boolean checkAndRowMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
399 ByteArrayComparable comparator, RowMutations mutations) throws IOException {
400 return checkAndRowMutate(row, family, qualifier, op, comparator, TimeRange.allTime(),
401 mutations);
405 * Atomically checks if a row/family/qualifier value matches the expected values and if it does,
406 * it performs the row mutations. If the passed value is null, the lack of column value
407 * (ie: non-existence) is used. Use to do many mutations on a single row. Use checkAndMutate
408 * to do one checkAndMutate at a time.
409 * @param row to check
410 * @param family column family to check
411 * @param qualifier column qualifier to check
412 * @param op the comparison operator
413 * @param comparator the expected value
414 * @param mutations data to put if check succeeds
415 * @param timeRange time range to check
416 * @return true if mutations were applied, false otherwise
418 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
419 * {@link #checkAndMutate(CheckAndMutate)} instead.
421 @Deprecated
422 boolean checkAndRowMutate(byte [] row, byte [] family, byte [] qualifier, CompareOperator op,
423 ByteArrayComparable comparator, TimeRange timeRange, RowMutations mutations)
424 throws IOException;
427 * Atomically checks if a row matches the filter and if it does, it performs the row mutations.
428 * Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a
429 * time.
430 * @param row to check
431 * @param filter the filter
432 * @param mutations data to put if check succeeds
433 * @return true if mutations were applied, false otherwise
435 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
436 * {@link #checkAndMutate(CheckAndMutate)} instead.
438 @Deprecated
439 default boolean checkAndRowMutate(byte[] row, Filter filter, RowMutations mutations)
440 throws IOException {
441 return checkAndRowMutate(row, filter, TimeRange.allTime(), mutations);
445 * Atomically checks if a row matches the filter and if it does, it performs the row mutations.
446 * Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a
447 * time.
448 * @param row to check
449 * @param filter the filter
450 * @param mutations data to put if check succeeds
451 * @param timeRange time range to check
452 * @return true if mutations were applied, false otherwise
454 * @deprecated since 3.0.0 and will be removed in 4.0.0. Use
455 * {@link #checkAndMutate(CheckAndMutate)} instead.
457 @Deprecated
458 boolean checkAndRowMutate(byte [] row, Filter filter, TimeRange timeRange,
459 RowMutations mutations) throws IOException;
462 * Atomically checks if a row matches the conditions and if it does, it performs the actions.
463 * Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a
464 * time.
465 * @param checkAndMutate the CheckAndMutate object
466 * @return true if mutations were applied, false otherwise
467 * @throws IOException if an error occurred in this method
469 CheckAndMutateResult checkAndMutate(CheckAndMutate checkAndMutate) throws IOException;
472 * Deletes the specified cells/row.
473 * @param delete
474 * @throws IOException
476 void delete(Delete delete) throws IOException;
479 * Do a get based on the get parameter.
480 * @param get query parameters
481 * @return result of the operation
483 Result get(Get get) throws IOException;
486 * Do a get based on the get parameter.
487 * @param get query parameters
488 * @param withCoprocessor invoke coprocessor or not. We don't want to
489 * always invoke cp.
490 * @return list of cells resulting from the operation
492 List<Cell> get(Get get, boolean withCoprocessor) throws IOException;
495 * Return an iterator that scans over the HRegion, returning the indicated
496 * columns and rows specified by the {@link Scan}.
497 * <p>
498 * This Iterator must be closed by the caller.
500 * @param scan configured {@link Scan}
501 * @return RegionScanner
502 * @throws IOException read exceptions
504 RegionScanner getScanner(Scan scan) throws IOException;
507 * Return an iterator that scans over the HRegion, returning the indicated columns and rows
508 * specified by the {@link Scan}. The scanner will also include the additional scanners passed
509 * along with the scanners for the specified Scan instance. Should be careful with the usage to
510 * pass additional scanners only within this Region
511 * <p>
512 * This Iterator must be closed by the caller.
514 * @param scan configured {@link Scan}
515 * @param additionalScanners Any additional scanners to be used
516 * @return RegionScanner
517 * @throws IOException read exceptions
519 RegionScanner getScanner(Scan scan, List<KeyValueScanner> additionalScanners) throws IOException;
521 /** The comparator to be used with the region */
522 CellComparator getCellComparator();
525 * Perform one or more increment operations on a row.
526 * @param increment
527 * @return result of the operation
528 * @throws IOException
530 Result increment(Increment increment) throws IOException;
533 * Performs multiple mutations atomically on a single row.
535 * @param mutations object that specifies the set of mutations to perform atomically
536 * @return results of Increment/Append operations. If no Increment/Append operations, it returns
537 * null
538 * @throws IOException
540 Result mutateRow(RowMutations mutations) throws IOException;
543 * Perform atomic mutations within the region.
545 * @param mutations The list of mutations to perform.
546 * <code>mutations</code> can contain operations for multiple rows.
547 * Caller has to ensure that all rows are contained in this region.
548 * @param rowsToLock Rows to lock
549 * @param nonceGroup Optional nonce group of the operation (client Id)
550 * @param nonce Optional nonce of the operation (unique random id to ensure "more idempotence")
551 * If multiple rows are locked care should be taken that
552 * <code>rowsToLock</code> is sorted in order to avoid deadlocks.
553 * @throws IOException
555 // TODO Should not be exposing with params nonceGroup, nonce. Change when doing the jira for
556 // Changing processRowsWithLocks
557 void mutateRowsWithLocks(Collection<Mutation> mutations, Collection<byte[]> rowsToLock,
558 long nonceGroup, long nonce) throws IOException;
561 * Puts some data in the table.
562 * @param put
563 * @throws IOException
565 void put(Put put) throws IOException;
567 ///////////////////////////////////////////////////////////////////////////
568 // Flushes, compactions, splits, etc.
569 // Wizards only, please
572 * @return if a given region is in compaction now.
574 CompactionState getCompactionState();
577 * Request compaction on this region.
579 void requestCompaction(String why, int priority, boolean major,
580 CompactionLifeCycleTracker tracker) throws IOException;
583 * Request compaction for the given family
585 void requestCompaction(byte[] family, String why, int priority, boolean major,
586 CompactionLifeCycleTracker tracker) throws IOException;
589 * Request flush on this region.
591 void requestFlush(FlushLifeCycleTracker tracker) throws IOException;
594 * Wait for all current flushes of the region to complete
596 * @param timeout The maximum time to wait in milliseconds.
597 * @return False when timeout elapsed but flushes are not over. True when flushes are over within
598 * max wait time period.
600 boolean waitForFlushes(long timeout);
603 * @return a read only configuration of this region; throws {@link UnsupportedOperationException}
604 * if you try to set a configuration.
606 Configuration getReadOnlyConfiguration();