001/*
002 * Licensed to the Apache Software Foundation (ASF) under one
003 * or more contributor license agreements.  See the NOTICE file
004 * distributed with this work for additional information
005 * regarding copyright ownership.  The ASF licenses this file
006 * to you under the Apache License, Version 2.0 (the
007 * "License"); you may not use this file except in compliance
008 * with the License.  You may obtain a copy of the License at
009 *
010 *     http://www.apache.org/licenses/LICENSE-2.0
011 *
012 * Unless required by applicable law or agreed to in writing, software
013 * distributed under the License is distributed on an "AS IS" BASIS,
014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015 * See the License for the specific language governing permissions and
016 * limitations under the License.
017 */
018package org.apache.hadoop.hbase.regionserver;
019
020import java.io.IOException;
021import java.util.Collection;
022import java.util.List;
023import java.util.Map;
024import org.apache.hadoop.conf.Configuration;
025import org.apache.hadoop.hbase.Cell;
026import org.apache.hadoop.hbase.CellComparator;
027import org.apache.hadoop.hbase.CompareOperator;
028import org.apache.hadoop.hbase.HBaseInterfaceAudience;
029import org.apache.hadoop.hbase.client.Append;
030import org.apache.hadoop.hbase.client.CheckAndMutate;
031import org.apache.hadoop.hbase.client.CheckAndMutateResult;
032import org.apache.hadoop.hbase.client.CompactionState;
033import org.apache.hadoop.hbase.client.Delete;
034import org.apache.hadoop.hbase.client.Get;
035import org.apache.hadoop.hbase.client.Increment;
036import org.apache.hadoop.hbase.client.Mutation;
037import org.apache.hadoop.hbase.client.Put;
038import org.apache.hadoop.hbase.client.RegionInfo;
039import org.apache.hadoop.hbase.client.Result;
040import org.apache.hadoop.hbase.client.RowMutations;
041import org.apache.hadoop.hbase.client.Scan;
042import org.apache.hadoop.hbase.client.TableDescriptor;
043import org.apache.hadoop.hbase.conf.ConfigurationObserver;
044import org.apache.hadoop.hbase.coprocessor.RegionCoprocessor;
045import org.apache.hadoop.hbase.filter.ByteArrayComparable;
046import org.apache.hadoop.hbase.filter.Filter;
047import org.apache.hadoop.hbase.io.TimeRange;
048import org.apache.hadoop.hbase.regionserver.compactions.CompactionLifeCycleTracker;
049import org.apache.yetus.audience.InterfaceAudience;
050import org.apache.yetus.audience.InterfaceStability;
051
052/**
053 * Region is a subset of HRegion with operations required for the {@link RegionCoprocessor
054 * Coprocessors}. The operations include ability to do mutations, requesting compaction, getting
055 * different counters/sizes, locking rows and getting access to {@linkplain Store}s.
056 */
057@InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.COPROC)
058@InterfaceStability.Evolving
059public interface Region extends ConfigurationObserver {
060
061  ///////////////////////////////////////////////////////////////////////////
062  // Region state
063
064  /** Returns region information for this region */
065  RegionInfo getRegionInfo();
066
067  /** Returns table descriptor for this region */
068  TableDescriptor getTableDescriptor();
069
070  /** Returns true if region is available (not closed and not closing) */
071  boolean isAvailable();
072
073  /** Returns true if region is closed */
074  boolean isClosed();
075
076  /** Returns True if closing process has started */
077  boolean isClosing();
078
079  /** Returns True if region is read only */
080  boolean isReadOnly();
081
082  /** Returns true if region is splittable */
083  boolean isSplittable();
084
085  /** Returns true if region is mergeable */
086  boolean isMergeable();
087
088  /**
089   * Return the list of Stores managed by this region
090   * <p>
091   * Use with caution. Exposed for use of fixup utilities.
092   * @return a list of the Stores managed by this region
093   */
094  List<? extends Store> getStores();
095
096  /**
097   * Return the Store for the given family
098   * <p>
099   * Use with caution. Exposed for use of fixup utilities.
100   * @return the Store for the given family
101   */
102  Store getStore(byte[] family);
103
104  /** Returns list of store file names for the given families */
105  List<String> getStoreFileList(byte[][] columns);
106
107  /**
108   * Check the region's underlying store files, open the files that have not been opened yet, and
109   * remove the store file readers for store files no longer available.
110   */
111  boolean refreshStoreFiles() throws IOException;
112
113  /**
114   * @return the max sequence id of flushed data on this region; no edit in memory will have a
115   *         sequence id that is less that what is returned here.
116   */
117  long getMaxFlushedSeqId();
118
119  /**
120   * This can be used to determine the last time all files of this region were major compacted.
121   * @param majorCompactionOnly Only consider HFile that are the result of major compaction
122   * @return the timestamp of the oldest HFile for all stores of this region
123   */
124  long getOldestHfileTs(boolean majorCompactionOnly) throws IOException;
125
126  /**
127   * @return map of column family names to max sequence id that was read from storage when this
128   *         region was opened
129   */
130  public Map<byte[], Long> getMaxStoreSeqId();
131
132  /**
133   * @return The earliest time a store in the region was flushed. All other stores in the region
134   *         would have been flushed either at, or after this time.
135   */
136  long getEarliestFlushTimeForAllStores();
137
138  ///////////////////////////////////////////////////////////////////////////
139  // Metrics
140
141  /** Returns read requests count for this region */
142  long getReadRequestsCount();
143
144  /** Returns filtered read requests count for this region */
145  long getFilteredReadRequestsCount();
146
147  /** Returns write request count for this region */
148  long getWriteRequestsCount();
149
150  /**
151   * @return memstore size for this region, in bytes. It just accounts data size of cells added to
152   *         the memstores of this Region. Means size in bytes for key, value and tags within Cells.
153   *         It wont consider any java heap overhead for the cell objects or any other.
154   */
155  long getMemStoreDataSize();
156
157  /**
158   * @return memstore heap size for this region, in bytes. It accounts data size of cells added to
159   *         the memstores of this Region, as well as java heap overhead for the cell objects or any
160   *         other.
161   */
162  long getMemStoreHeapSize();
163
164  /**
165   * @return memstore off-heap size for this region, in bytes. It accounts data size of cells added
166   *         to the memstores of this Region, as well as overhead for the cell objects or any other
167   *         that is allocated off-heap.
168   */
169  long getMemStoreOffHeapSize();
170
171  /** Returns the number of mutations processed bypassing the WAL */
172  long getNumMutationsWithoutWAL();
173
174  /** Returns the size of data processed bypassing the WAL, in bytes */
175  long getDataInMemoryWithoutWAL();
176
177  /** Returns the number of blocked requests */
178  long getBlockedRequestsCount();
179
180  /** Returns the number of checkAndMutate guards that passed */
181  long getCheckAndMutateChecksPassed();
182
183  /** Returns the number of failed checkAndMutate guards */
184  long getCheckAndMutateChecksFailed();
185
186  ///////////////////////////////////////////////////////////////////////////
187  // Locking
188
189  // Region read locks
190
191  /**
192   * Operation enum is used in {@link Region#startRegionOperation} and elsewhere to provide context
193   * for various checks.
194   */
195  enum Operation {
196    ANY,
197    GET,
198    PUT,
199    DELETE,
200    SCAN,
201    APPEND,
202    INCREMENT,
203    SPLIT_REGION,
204    MERGE_REGION,
205    BATCH_MUTATE,
206    REPLAY_BATCH_MUTATE,
207    COMPACT_REGION,
208    REPLAY_EVENT,
209    SNAPSHOT,
210    COMPACT_SWITCH,
211    CHECK_AND_MUTATE
212  }
213
214  /**
215   * This method needs to be called before any public call that reads or modifies data. Acquires a
216   * read lock and checks if the region is closing or closed.
217   * <p>
218   * {@link #closeRegionOperation} MUST then always be called after the operation has completed,
219   * whether it succeeded or failed.
220   */
221  // TODO Exposing this and closeRegionOperation() as we have getRowLock() exposed.
222  // Remove if we get rid of exposing getRowLock().
223  void startRegionOperation() throws IOException;
224
225  /**
226   * This method needs to be called before any public call that reads or modifies data. Acquires a
227   * read lock and checks if the region is closing or closed.
228   * <p>
229   * {@link #closeRegionOperation} MUST then always be called after the operation has completed,
230   * whether it succeeded or failed.
231   * @param op The operation is about to be taken on the region
232   */
233  void startRegionOperation(Operation op) throws IOException;
234
235  /**
236   * Closes the region operation lock.
237   */
238  void closeRegionOperation() throws IOException;
239
240  /**
241   * Closes the region operation lock. This needs to be called in the finally block corresponding to
242   * the try block of {@link #startRegionOperation(Operation)}
243   */
244  void closeRegionOperation(Operation op) throws IOException;
245
246  // Row write locks
247
248  /**
249   * Row lock held by a given thread. One thread may acquire multiple locks on the same row
250   * simultaneously. The locks must be released by calling release() from the same thread.
251   */
252  public interface RowLock {
253    /**
254     * Release the given lock. If there are no remaining locks held by the current thread then
255     * unlock the row and allow other threads to acquire the lock.
256     * @throws IllegalArgumentException if called by a different thread than the lock owning thread
257     */
258    void release();
259  }
260
261  /**
262   * Get a row lock for the specified row. All locks are reentrant. Before calling this function
263   * make sure that a region operation has already been started (the calling thread has already
264   * 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 stated whether or not
269   * to wait on the lock. Now it is whether it an exclusive lock is requested.
270   * @param row      The row actions will be performed against
271   * @param readLock is the lock reader or writer. True indicates that a non-exclusive lock is
272   *                 requested
273   * @see #startRegionOperation()
274   * @see #startRegionOperation(Operation)
275   */
276  // TODO this needs to be exposed as we have RowProcessor now. If RowProcessor is removed, we can
277  // remove this too..
278  RowLock getRowLock(byte[] row, boolean readLock) throws IOException;
279
280  ///////////////////////////////////////////////////////////////////////////
281  // Region operations
282
283  /**
284   * Perform one or more append operations on a row.
285   * @return result of the operation
286   */
287  Result append(Append append) throws IOException;
288
289  /**
290   * Perform a batch of mutations.
291   * <p>
292   * Please do not operate on a same column of a single row in a batch, we will not consider the
293   * previous operation in the same batch when performing the operations in the batch.
294   * @param mutations the list of mutations
295   * @return an array of OperationStatus which internally contains the OperationStatusCode and the
296   *         exceptionMessage if any.
297   */
298  OperationStatus[] batchMutate(Mutation[] mutations) throws IOException;
299
300  /**
301   * Atomically checks if a row/family/qualifier value matches the expected value and if it does, it
302   * performs the mutation. If the passed value is null, the lack of column value (ie:
303   * non-existence) is used. See checkAndRowMutate to do many checkAndPuts at a time on a single
304   * row.
305   * @param row        to check
306   * @param family     column family to check
307   * @param qualifier  column qualifier to check
308   * @param op         the comparison operator
309   * @param comparator the expected value
310   * @param mutation   data to put if check succeeds
311   * @return true if mutation was applied, false otherwise
312   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
313   *             {@link #checkAndMutate(CheckAndMutate)} instead.
314   */
315  @Deprecated
316  default boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
317    ByteArrayComparable comparator, Mutation mutation) throws IOException {
318    return checkAndMutate(row, family, qualifier, op, comparator, TimeRange.allTime(), mutation);
319  }
320
321  /**
322   * Atomically checks if a row/family/qualifier value matches the expected value and if it does, it
323   * performs the mutation. If the passed value is null, the lack of column value (ie:
324   * non-existence) is used. See checkAndRowMutate to do many checkAndPuts at a time on a single
325   * row.
326   * @param row        to check
327   * @param family     column family to check
328   * @param qualifier  column qualifier to check
329   * @param op         the comparison operator
330   * @param comparator the expected value
331   * @param mutation   data to put if check succeeds
332   * @param timeRange  time range to check
333   * @return true if mutation was applied, false otherwise
334   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
335   *             {@link #checkAndMutate(CheckAndMutate)} instead.
336   */
337  @Deprecated
338  boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
339    ByteArrayComparable comparator, TimeRange timeRange, Mutation mutation) throws IOException;
340
341  /**
342   * Atomically checks if a row matches the filter and if it does, it performs the mutation. See
343   * checkAndRowMutate to do many checkAndPuts at a time on a single row.
344   * @param row      to check
345   * @param filter   the filter
346   * @param mutation data to put if check succeeds
347   * @return true if mutation was applied, false otherwise
348   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
349   *             {@link #checkAndMutate(CheckAndMutate)} instead.
350   */
351  @Deprecated
352  default boolean checkAndMutate(byte[] row, Filter filter, Mutation mutation) throws IOException {
353    return checkAndMutate(row, filter, TimeRange.allTime(), mutation);
354  }
355
356  /**
357   * Atomically checks if a row value matches the filter and if it does, it performs the mutation.
358   * See checkAndRowMutate to do many checkAndPuts at a time on a single row.
359   * @param row       to check
360   * @param filter    the filter
361   * @param mutation  data to put if check succeeds
362   * @param timeRange time range to check
363   * @return true if mutation was applied, false otherwise
364   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
365   *             {@link #checkAndMutate(CheckAndMutate)} instead.
366   */
367  @Deprecated
368  boolean checkAndMutate(byte[] row, Filter filter, TimeRange timeRange, Mutation mutation)
369    throws IOException;
370
371  /**
372   * Atomically checks if a row/family/qualifier value matches the expected values and if it does,
373   * it performs the row mutations. If the passed value is null, the lack of column value (ie:
374   * non-existence) is used. Use to do many mutations on a single row. Use checkAndMutate to do one
375   * checkAndMutate at a time.
376   * @param row        to check
377   * @param family     column family to check
378   * @param qualifier  column qualifier to check
379   * @param op         the comparison operator
380   * @param comparator the expected value
381   * @param mutations  data to put if check succeeds
382   * @return true if mutations were applied, false otherwise
383   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
384   *             {@link #checkAndMutate(CheckAndMutate)} instead.
385   */
386  @Deprecated
387  default boolean checkAndRowMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
388    ByteArrayComparable comparator, RowMutations mutations) throws IOException {
389    return checkAndRowMutate(row, family, qualifier, op, comparator, TimeRange.allTime(),
390      mutations);
391  }
392
393  /**
394   * Atomically checks if a row/family/qualifier value matches the expected values and if it does,
395   * it performs the row mutations. If the passed value is null, the lack of column value (ie:
396   * non-existence) is used. Use to do many mutations on a single row. Use checkAndMutate to do one
397   * checkAndMutate at a time.
398   * @param row        to check
399   * @param family     column family to check
400   * @param qualifier  column qualifier to check
401   * @param op         the comparison operator
402   * @param comparator the expected value
403   * @param mutations  data to put if check succeeds
404   * @param timeRange  time range to check
405   * @return true if mutations were applied, false otherwise
406   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
407   *             {@link #checkAndMutate(CheckAndMutate)} instead.
408   */
409  @Deprecated
410  boolean checkAndRowMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
411    ByteArrayComparable comparator, TimeRange timeRange, RowMutations mutations) throws IOException;
412
413  /**
414   * Atomically checks if a row matches the filter and if it does, it performs the row mutations.
415   * Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a
416   * time.
417   * @param row       to check
418   * @param filter    the filter
419   * @param mutations data to put if check succeeds
420   * @return true if mutations were applied, false otherwise
421   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
422   *             {@link #checkAndMutate(CheckAndMutate)} instead.
423   */
424  @Deprecated
425  default boolean checkAndRowMutate(byte[] row, Filter filter, RowMutations mutations)
426    throws IOException {
427    return checkAndRowMutate(row, filter, TimeRange.allTime(), mutations);
428  }
429
430  /**
431   * Atomically checks if a row matches the filter and if it does, it performs the row mutations.
432   * Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a
433   * time.
434   * @param row       to check
435   * @param filter    the filter
436   * @param mutations data to put if check succeeds
437   * @param timeRange time range to check
438   * @return true if mutations were applied, false otherwise
439   * @deprecated since 2.4.0 and will be removed in 4.0.0. Use
440   *             {@link #checkAndMutate(CheckAndMutate)} instead.
441   */
442  @Deprecated
443  boolean checkAndRowMutate(byte[] row, Filter filter, TimeRange timeRange, RowMutations mutations)
444    throws IOException;
445
446  /**
447   * Atomically checks if a row matches the conditions and if it does, it performs the actions. Use
448   * to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a time.
449   * @param checkAndMutate the CheckAndMutate object
450   * @return true if mutations were applied, false otherwise
451   * @throws IOException if an error occurred in this method
452   */
453  CheckAndMutateResult checkAndMutate(CheckAndMutate checkAndMutate) throws IOException;
454
455  /**
456   * Deletes the specified cells/row.
457   */
458  void delete(Delete delete) throws IOException;
459
460  /**
461   * Do a get based on the get parameter.
462   * @param get query parameters
463   * @return result of the operation
464   */
465  Result get(Get get) throws IOException;
466
467  /**
468   * Do a get based on the get parameter.
469   * @param get             query parameters
470   * @param withCoprocessor invoke coprocessor or not. We don't want to always invoke cp.
471   * @return list of cells resulting from the operation
472   */
473  List<Cell> get(Get get, boolean withCoprocessor) throws IOException;
474
475  /**
476   * Return an iterator that scans over the HRegion, returning the indicated columns and rows
477   * specified by the {@link Scan}.
478   * <p>
479   * This Iterator must be closed by the caller.
480   * @param scan configured {@link Scan}
481   * @throws IOException read exceptions
482   */
483  RegionScanner getScanner(Scan scan) throws IOException;
484
485  /**
486   * Return an iterator that scans over the HRegion, returning the indicated columns and rows
487   * specified by the {@link Scan}. The scanner will also include the additional scanners passed
488   * along with the scanners for the specified Scan instance. Should be careful with the usage to
489   * pass additional scanners only within this Region
490   * <p>
491   * This Iterator must be closed by the caller.
492   * @param scan               configured {@link Scan}
493   * @param additionalScanners Any additional scanners to be used
494   * @throws IOException read exceptions
495   */
496  RegionScanner getScanner(Scan scan, List<KeyValueScanner> additionalScanners) throws IOException;
497
498  /** The comparator to be used with the region */
499  CellComparator getCellComparator();
500
501  /**
502   * Perform one or more increment operations on a row.
503   * @return result of the operation
504   */
505  Result increment(Increment increment) throws IOException;
506
507  /**
508   * Performs multiple mutations atomically on a single row.
509   * @param mutations object that specifies the set of mutations to perform atomically
510   * @return results of Increment/Append operations. If no Increment/Append operations, it returns
511   *         null
512   */
513  Result mutateRow(RowMutations mutations) throws IOException;
514
515  /**
516   * Perform atomic mutations within the region.
517   * @param mutations  The list of mutations to perform. <code>mutations</code> can contain
518   *                   operations for multiple rows. Caller has to ensure that all rows are
519   *                   contained in this region.
520   * @param rowsToLock Rows to lock
521   * @param nonceGroup Optional nonce group of the operation (client Id)
522   * @param nonce      Optional nonce of the operation (unique random id to ensure "more
523   *                   idempotence") If multiple rows are locked care should be taken that
524   *                   <code>rowsToLock</code> is sorted in order to avoid deadlocks.
525   */
526  // TODO Should not be exposing with params nonceGroup, nonce. Change when doing the jira for
527  // Changing processRowsWithLocks and RowProcessor
528  void mutateRowsWithLocks(Collection<Mutation> mutations, Collection<byte[]> rowsToLock,
529    long nonceGroup, long nonce) throws IOException;
530
531  /**
532   * Performs atomic multiple reads and writes on a given row.
533   * @param processor The object defines the reads and writes to a row.
534   * @deprecated As of release 2.0.0, this will be removed in HBase 3.0.0. For customization, use
535   *             Coprocessors instead.
536   */
537  @Deprecated
538  void processRowsWithLocks(RowProcessor<?, ?> processor) throws IOException;
539
540  /**
541   * Performs atomic multiple reads and writes on a given row.
542   * @param processor  The object defines the reads and writes to a row.
543   * @param nonceGroup Optional nonce group of the operation (client Id)
544   * @param nonce      Optional nonce of the operation (unique random id to ensure "more
545   *                   idempotence")
546   * @deprecated As of release 2.0.0, this will be removed in HBase 3.0.0. For customization, use
547   *             Coprocessors instead.
548   */
549  // TODO Should not be exposing with params nonceGroup, nonce. Change when doing the jira for
550  // Changing processRowsWithLocks and RowProcessor
551  @Deprecated
552  void processRowsWithLocks(RowProcessor<?, ?> processor, long nonceGroup, long nonce)
553    throws IOException;
554
555  /**
556   * Performs atomic multiple reads and writes on a given row.
557   * @param processor  The object defines the reads and writes to a row.
558   * @param timeout    The timeout of the processor.process() execution Use a negative number to
559   *                   switch off the time bound
560   * @param nonceGroup Optional nonce group of the operation (client Id)
561   * @param nonce      Optional nonce of the operation (unique random id to ensure "more
562   *                   idempotence")
563   * @deprecated As of release 2.0.0, this will be removed in HBase 3.0.0. For customization, use
564   *             Coprocessors instead.
565   */
566  // TODO Should not be exposing with params nonceGroup, nonce. Change when doing the jira for
567  // Changing processRowsWithLocks and RowProcessor
568  @Deprecated
569  void processRowsWithLocks(RowProcessor<?, ?> processor, long timeout, long nonceGroup, long nonce)
570    throws IOException;
571
572  /**
573   * Puts some data in the table.
574   */
575  void put(Put put) throws IOException;
576
577  ///////////////////////////////////////////////////////////////////////////
578  // Flushes, compactions, splits, etc.
579  // Wizards only, please
580
581  /** Returns if a given region is in compaction now. */
582  CompactionState getCompactionState();
583
584  /**
585   * Request compaction on this region.
586   */
587  void requestCompaction(String why, int priority, boolean major,
588    CompactionLifeCycleTracker tracker) throws IOException;
589
590  /**
591   * Request compaction for the given family
592   */
593  void requestCompaction(byte[] family, String why, int priority, boolean major,
594    CompactionLifeCycleTracker tracker) throws IOException;
595
596  /**
597   * Request flush on this region.
598   */
599  void requestFlush(FlushLifeCycleTracker tracker) throws IOException;
600
601  /**
602   * Wait for all current flushes of the region to complete
603   * @param timeout The maximum time to wait in milliseconds.
604   * @return False when timeout elapsed but flushes are not over. True when flushes are over within
605   *         max wait time period.
606   */
607  boolean waitForFlushes(long timeout);
608
609  /**
610   * @return a read only configuration of this region; throws {@link UnsupportedOperationException}
611   *         if you try to set a configuration.
612   */
613  Configuration getReadOnlyConfiguration();
614
615  /**
616   * The minimum block size configuration from all relevant column families. This is used when
617   * estimating quota consumption.
618   */
619  int getMinBlockSizeBytes();
620}