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,
013 * software distributed under the License is distributed on an
014 * "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
015 * KIND, either express or implied.  See the License for the
016 * specific language governing permissions and limitations
017 * under the License.
018 */
019
020package org.apache.hadoop.hbase.coprocessor;
021
022import java.io.IOException;
023import java.util.ArrayList;
024import java.util.List;
025import java.util.Map;
026
027import org.apache.hadoop.fs.FileSystem;
028import org.apache.hadoop.fs.Path;
029import org.apache.hadoop.hbase.Cell;
030import org.apache.hadoop.hbase.CompareOperator;
031import org.apache.hadoop.hbase.HBaseInterfaceAudience;
032import org.apache.hadoop.hbase.client.Append;
033import org.apache.hadoop.hbase.client.Delete;
034import org.apache.hadoop.hbase.client.Durability;
035import org.apache.hadoop.hbase.client.Get;
036import org.apache.hadoop.hbase.client.Increment;
037import org.apache.hadoop.hbase.client.Mutation;
038import org.apache.hadoop.hbase.client.Put;
039import org.apache.hadoop.hbase.client.RegionInfo;
040import org.apache.hadoop.hbase.client.Result;
041import org.apache.hadoop.hbase.client.Scan;
042import org.apache.hadoop.hbase.filter.ByteArrayComparable;
043import org.apache.hadoop.hbase.io.FSDataInputStreamWrapper;
044import org.apache.hadoop.hbase.io.Reference;
045import org.apache.hadoop.hbase.io.hfile.CacheConfig;
046import org.apache.hadoop.hbase.regionserver.FlushLifeCycleTracker;
047import org.apache.hadoop.hbase.regionserver.InternalScanner;
048import org.apache.hadoop.hbase.regionserver.MiniBatchOperationInProgress;
049import org.apache.hadoop.hbase.regionserver.OperationStatus;
050import org.apache.hadoop.hbase.regionserver.Region;
051import org.apache.hadoop.hbase.regionserver.Region.Operation;
052import org.apache.hadoop.hbase.regionserver.RegionScanner;
053import org.apache.hadoop.hbase.regionserver.ScanOptions;
054import org.apache.hadoop.hbase.regionserver.ScanType;
055import org.apache.hadoop.hbase.regionserver.Store;
056import org.apache.hadoop.hbase.regionserver.StoreFile;
057import org.apache.hadoop.hbase.regionserver.StoreFileReader;
058import org.apache.hadoop.hbase.regionserver.compactions.CompactionLifeCycleTracker;
059import org.apache.hadoop.hbase.regionserver.compactions.CompactionRequest;
060import org.apache.hadoop.hbase.regionserver.querymatcher.DeleteTracker;
061import org.apache.hadoop.hbase.util.Pair;
062import org.apache.hadoop.hbase.wal.WALEdit;
063import org.apache.hadoop.hbase.wal.WALKey;
064import org.apache.yetus.audience.InterfaceAudience;
065import org.apache.yetus.audience.InterfaceStability;
066
067/**
068 * Coprocessors implement this interface to observe and mediate client actions on the region.
069 * <p>
070 * Since most implementations will be interested in only a subset of hooks, this class uses
071 * 'default' functions to avoid having to add unnecessary overrides. When the functions are
072 * non-empty, it's simply to satisfy the compiler by returning value of expected (non-void) type. It
073 * is done in a way that these default definitions act as no-op. So our suggestion to implementation
074 * would be to not call these 'default' methods from overrides.
075 * <p>
076 * <h3>Exception Handling</h3><br>
077 * For all functions, exception handling is done as follows:
078 * <ul>
079 * <li>Exceptions of type {@link IOException} are reported back to client.</li>
080 * <li>For any other kind of exception:
081 * <ul>
082 * <li>If the configuration {@link CoprocessorHost#ABORT_ON_ERROR_KEY} is set to true, then the
083 * server aborts.</li>
084 * <li>Otherwise, coprocessor is removed from the server and
085 * {@link org.apache.hadoop.hbase.DoNotRetryIOException} is returned to the client.</li>
086 * </ul>
087 * </li>
088 * </ul>
089 * <p>
090 * <h3>For Split Related Hooks</h3> <br>
091 * In hbase2/AMv2, master runs splits, so the split related hooks are moved to
092 * {@link MasterObserver}.
093 * <p>
094 * <h3>Increment Column Value</h3><br>
095 * We do not call this hook anymore.
096 */
097@InterfaceAudience.LimitedPrivate(HBaseInterfaceAudience.COPROC)
098@InterfaceStability.Evolving
099// TODO as method signatures need to break, update to
100// ObserverContext<? extends RegionCoprocessorEnvironment>
101// so we can use additional environment state that isn't exposed to coprocessors.
102public interface RegionObserver {
103  /** Mutation type for postMutationBeforeWAL hook */
104  enum MutationType {
105    APPEND, INCREMENT
106  }
107
108  /**
109   * Called before the region is reported as open to the master.
110   * @param c the environment provided by the region server
111   */
112  default void preOpen(ObserverContext<RegionCoprocessorEnvironment> c) throws IOException {}
113
114  /**
115   * Called after the region is reported as open to the master.
116   * @param c the environment provided by the region server
117   */
118  default void postOpen(ObserverContext<RegionCoprocessorEnvironment> c) {}
119
120  /**
121   * Called before the memstore is flushed to disk.
122   * @param c the environment provided by the region server
123   * @param tracker tracker used to track the life cycle of a flush
124   */
125  default void preFlush(final ObserverContext<RegionCoprocessorEnvironment> c,
126      FlushLifeCycleTracker tracker) throws IOException {}
127
128  /**
129   * Called before we open store scanner for flush. You can use the {@code options} to change max
130   * versions and TTL for the scanner being opened.
131   * @param c the environment provided by the region server
132   * @param store the store where flush is being requested
133   * @param options used to change max versions and TTL for the scanner being opened
134   */
135  default void preFlushScannerOpen(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
136      ScanOptions options,FlushLifeCycleTracker tracker) throws IOException {}
137
138  /**
139   * Called before a Store's memstore is flushed to disk.
140   * @param c the environment provided by the region server
141   * @param store the store where flush is being requested
142   * @param scanner the scanner over existing data used in the memstore
143   * @param tracker tracker used to track the life cycle of a flush
144   * @return the scanner to use during flush. Should not be {@code null} unless the implementation
145   *         is writing new store files on its own.
146   */
147  default InternalScanner preFlush(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
148      InternalScanner scanner, FlushLifeCycleTracker tracker) throws IOException {
149    return scanner;
150  }
151
152  /**
153   * Called after the memstore is flushed to disk.
154   * @param c the environment provided by the region server
155   * @param tracker tracker used to track the life cycle of a flush
156   * @throws IOException if an error occurred on the coprocessor
157   */
158  default void postFlush(ObserverContext<RegionCoprocessorEnvironment> c,
159      FlushLifeCycleTracker tracker) throws IOException {}
160
161  /**
162   * Called after a Store's memstore is flushed to disk.
163   * @param c the environment provided by the region server
164   * @param store the store being flushed
165   * @param resultFile the new store file written out during compaction
166   * @param tracker tracker used to track the life cycle of a flush
167   */
168  default void postFlush(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
169      StoreFile resultFile, FlushLifeCycleTracker tracker) throws IOException {}
170
171  /**
172   * Called before in memory compaction started.
173   * @param c the environment provided by the region server
174   * @param store the store where in memory compaction is being requested
175   */
176  default void preMemStoreCompaction(ObserverContext<RegionCoprocessorEnvironment> c, Store store)
177      throws IOException {}
178
179  /**
180   * Called before we open store scanner for in memory compaction. You can use the {@code options}
181   * to change max versions and TTL for the scanner being opened. Notice that this method will only
182   * be called when you use {@code eager} mode. For {@code basic} mode we will not drop any cells
183   * thus we do not open a store scanner.
184   * @param c the environment provided by the region server
185   * @param store the store where in memory compaction is being requested
186   * @param options used to change max versions and TTL for the scanner being opened
187   */
188  default void preMemStoreCompactionCompactScannerOpen(
189      ObserverContext<RegionCoprocessorEnvironment> c, Store store, ScanOptions options)
190      throws IOException {}
191
192  /**
193   * Called before we do in memory compaction. Notice that this method will only be called when you
194   * use {@code eager} mode. For {@code basic} mode we will not drop any cells thus there is no
195   * {@link InternalScanner}.
196   * @param c the environment provided by the region server
197   * @param store the store where in memory compaction is being executed
198   * @param scanner the scanner over existing data used in the memstore segments being compact
199   * @return the scanner to use during in memory compaction. Must be non-null.
200   */
201  default InternalScanner preMemStoreCompactionCompact(
202      ObserverContext<RegionCoprocessorEnvironment> c, Store store, InternalScanner scanner)
203      throws IOException {
204    return scanner;
205  }
206
207  /**
208   * Called after the in memory compaction is finished.
209   * @param c the environment provided by the region server
210   * @param store the store where in memory compaction is being executed
211   */
212  default void postMemStoreCompaction(ObserverContext<RegionCoprocessorEnvironment> c, Store store)
213      throws IOException {}
214
215  /**
216   * Called prior to selecting the {@link StoreFile StoreFiles} to compact from the list of
217   * available candidates. To alter the files used for compaction, you may mutate the passed in list
218   * of candidates. If you remove all the candidates then the compaction will be canceled.
219   * <p>Supports Coprocessor 'bypass' -- 'bypass' is how this method indicates that it changed
220   * the passed in <code>candidates</code>.
221   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
222   * @param c the environment provided by the region server
223   * @param store the store where compaction is being requested
224   * @param candidates the store files currently available for compaction
225   * @param tracker tracker used to track the life cycle of a compaction
226   */
227  default void preCompactSelection(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
228      List<? extends StoreFile> candidates, CompactionLifeCycleTracker tracker)
229      throws IOException {}
230
231  /**
232   * Called after the {@link StoreFile}s to compact have been selected from the available
233   * candidates.
234   * @param c the environment provided by the region server
235   * @param store the store being compacted
236   * @param selected the store files selected to compact
237   * @param tracker tracker used to track the life cycle of a compaction
238   * @param request the requested compaction
239   */
240  default void postCompactSelection(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
241      List<? extends StoreFile> selected, CompactionLifeCycleTracker tracker,
242      CompactionRequest request) {}
243
244  /**
245   * Called before we open store scanner for compaction. You can use the {@code options} to change max
246   * versions and TTL for the scanner being opened.
247   * @param c the environment provided by the region server
248   * @param store the store being compacted
249   * @param scanType type of Scan
250   * @param options used to change max versions and TTL for the scanner being opened
251   * @param tracker tracker used to track the life cycle of a compaction
252   * @param request the requested compaction
253   */
254  default void preCompactScannerOpen(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
255      ScanType scanType, ScanOptions options, CompactionLifeCycleTracker tracker,
256      CompactionRequest request) throws IOException {}
257
258  /**
259   * Called prior to writing the {@link StoreFile}s selected for compaction into a new
260   * {@code StoreFile}.
261   * <p>
262   * To override or modify the compaction process, implementing classes can wrap the provided
263   * {@link InternalScanner} with a custom implementation that is returned from this method. The
264   * custom scanner can then inspect {@link org.apache.hadoop.hbase.Cell}s from the wrapped scanner,
265   * applying its own policy to what gets written.
266   * @param c the environment provided by the region server
267   * @param store the store being compacted
268   * @param scanner the scanner over existing data used in the store file rewriting
269   * @param scanType type of Scan
270   * @param tracker tracker used to track the life cycle of a compaction
271   * @param request the requested compaction
272   * @return the scanner to use during compaction. Should not be {@code null} unless the
273   *         implementation is writing new store files on its own.
274   */
275  default InternalScanner preCompact(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
276      InternalScanner scanner, ScanType scanType, CompactionLifeCycleTracker tracker,
277      CompactionRequest request) throws IOException {
278    return scanner;
279  }
280
281  /**
282   * Called after compaction has completed and the new store file has been moved in to place.
283   * @param c the environment provided by the region server
284   * @param store the store being compacted
285   * @param resultFile the new store file written out during compaction
286   * @param tracker used to track the life cycle of a compaction
287   * @param request the requested compaction
288   */
289  default void postCompact(ObserverContext<RegionCoprocessorEnvironment> c, Store store,
290      StoreFile resultFile, CompactionLifeCycleTracker tracker, CompactionRequest request)
291      throws IOException {}
292
293  /**
294   * Called before the region is reported as closed to the master.
295   * @param c the environment provided by the region server
296   * @param abortRequested true if the region server is aborting
297   */
298  default void preClose(ObserverContext<RegionCoprocessorEnvironment> c, boolean abortRequested)
299      throws IOException {}
300
301  /**
302   * Called after the region is reported as closed to the master.
303   * @param c the environment provided by the region server
304   * @param abortRequested true if the region server is aborting
305   */
306  default void postClose(ObserverContext<RegionCoprocessorEnvironment> c, boolean abortRequested) {}
307
308  /**
309   * Called before the client performs a Get
310   * <p>
311   * Call CoprocessorEnvironment#bypass to skip default actions.
312   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
313   * @param c the environment provided by the region server
314   * @param get the Get request
315   * @param result The result to return to the client if default processing
316   * is bypassed. Can be modified. Will not be used if default processing
317   * is not bypassed.
318   */
319  default void preGetOp(ObserverContext<RegionCoprocessorEnvironment> c, Get get, List<Cell> result)
320      throws IOException {}
321
322  /**
323   * Called after the client performs a Get
324   * <p>
325   * Note: Do not retain references to any Cells in 'result' beyond the life of this invocation.
326   * If need a Cell reference for later use, copy the cell and use that.
327   * @param c the environment provided by the region server
328   * @param get the Get request
329   * @param result the result to return to the client, modify as necessary
330   */
331  default void postGetOp(ObserverContext<RegionCoprocessorEnvironment> c, Get get,
332      List<Cell> result) throws IOException {}
333
334  /**
335   * Called before the client tests for existence using a Get.
336   * <p>
337   * Call CoprocessorEnvironment#bypass to skip default actions.
338   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
339   * @param c the environment provided by the region server
340   * @param get the Get request
341   * @param exists the result returned by the region server
342   * @return the value to return to the client if bypassing default processing
343   */
344  default boolean preExists(ObserverContext<RegionCoprocessorEnvironment> c, Get get,
345      boolean exists) throws IOException {
346    return exists;
347  }
348
349  /**
350   * Called after the client tests for existence using a Get.
351   * @param c the environment provided by the region server
352   * @param get the Get request
353   * @param exists the result returned by the region server
354   * @return the result to return to the client
355   */
356  default boolean postExists(ObserverContext<RegionCoprocessorEnvironment> c, Get get,
357      boolean exists) throws IOException {
358    return exists;
359  }
360
361  /**
362   * Called before the client stores a value.
363   * <p>
364   * Call CoprocessorEnvironment#bypass to skip default actions.
365   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
366   * <p>
367   * Note: Do not retain references to any Cells in 'put' beyond the life of this invocation.
368   * If need a Cell reference for later use, copy the cell and use that.
369   * @param c the environment provided by the region server
370   * @param put The Put object
371   * @param edit The WALEdit object that will be written to the wal
372   * @param durability Persistence guarantee for this Put
373   */
374  default void prePut(ObserverContext<RegionCoprocessorEnvironment> c, Put put, WALEdit edit,
375      Durability durability) throws IOException {}
376
377  /**
378   * Called after the client stores a value.
379   * <p>
380   * Note: Do not retain references to any Cells in 'put' beyond the life of this invocation.
381   * If need a Cell reference for later use, copy the cell and use that.
382   * @param c the environment provided by the region server
383   * @param put The Put object
384   * @param edit The WALEdit object for the wal
385   * @param durability Persistence guarantee for this Put
386   */
387  default void postPut(ObserverContext<RegionCoprocessorEnvironment> c, Put put, WALEdit edit,
388      Durability durability) throws IOException {}
389
390  /**
391   * Called before the client deletes a value.
392   * <p>
393   * Call CoprocessorEnvironment#bypass to skip default actions.
394   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
395   * <p>
396   * Note: Do not retain references to any Cells in 'delete' beyond the life of this invocation.
397   * If need a Cell reference for later use, copy the cell and use that.
398   * @param c the environment provided by the region server
399   * @param delete The Delete object
400   * @param edit The WALEdit object for the wal
401   * @param durability Persistence guarantee for this Delete
402   */
403  default void preDelete(ObserverContext<RegionCoprocessorEnvironment> c, Delete delete,
404      WALEdit edit, Durability durability) throws IOException {}
405
406  /**
407   * Called before the server updates the timestamp for version delete with latest timestamp.
408   * <p>
409   * Call CoprocessorEnvironment#bypass to skip default actions.
410   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
411   * @param c the environment provided by the region server
412   * @param mutation - the parent mutation associated with this delete cell
413   * @param cell - The deleteColumn with latest version cell
414   * @param byteNow - timestamp bytes
415   * @param get - the get formed using the current cell's row. Note that the get does not specify
416   *          the family and qualifier
417   * @deprecated Since hbase-2.0.0. No replacement. To be removed in hbase-3.0.0 and replaced
418   * with something that doesn't expose IntefaceAudience.Private classes.
419   */
420  @Deprecated
421  default void prePrepareTimeStampForDeleteVersion(ObserverContext<RegionCoprocessorEnvironment> c,
422      Mutation mutation, Cell cell, byte[] byteNow, Get get) throws IOException {}
423
424  /**
425   * Called after the client deletes a value.
426   * <p>
427   * Note: Do not retain references to any Cells in 'delete' beyond the life of this invocation.
428   * If need a Cell reference for later use, copy the cell and use that.
429   * @param c the environment provided by the region server
430   * @param delete The Delete object
431   * @param edit The WALEdit object for the wal
432   * @param durability Persistence guarantee for this Delete
433   */
434  default void postDelete(ObserverContext<RegionCoprocessorEnvironment> c, Delete delete,
435      WALEdit edit, Durability durability) throws IOException {}
436
437  /**
438   * This will be called for every batch mutation operation happening at the server. This will be
439   * called after acquiring the locks on the mutating rows and after applying the proper timestamp
440   * for each Mutation at the server. The batch may contain Put/Delete. By setting OperationStatus
441   * of Mutations ({@link MiniBatchOperationInProgress#setOperationStatus(int, OperationStatus)}),
442   * {@link RegionObserver} can make Region to skip these Mutations.
443   * <p>
444   * Note: Do not retain references to any Cells in Mutations beyond the life of this invocation.
445   * If need a Cell reference for later use, copy the cell and use that.
446   * @param c the environment provided by the region server
447   * @param miniBatchOp batch of Mutations getting applied to region.
448   */
449  default void preBatchMutate(ObserverContext<RegionCoprocessorEnvironment> c,
450      MiniBatchOperationInProgress<Mutation> miniBatchOp) throws IOException {}
451
452  /**
453   * This will be called after applying a batch of Mutations on a region. The Mutations are added to
454   * memstore and WAL. The difference of this one with
455   * {@link #postPut(ObserverContext, Put, WALEdit, Durability) }
456   * and {@link #postDelete(ObserverContext, Delete, WALEdit, Durability) } is
457   * this hook will be executed before the mvcc transaction completion.
458   * <p>
459   * Note: Do not retain references to any Cells in Mutations beyond the life of this invocation.
460   * If need a Cell reference for later use, copy the cell and use that.
461   * @param c the environment provided by the region server
462   * @param miniBatchOp batch of Mutations applied to region. Coprocessors are discouraged from
463   *                    manipulating its state.
464   */
465  // Coprocessors can do a form of bypass by changing state in miniBatchOp.
466  default void postBatchMutate(ObserverContext<RegionCoprocessorEnvironment> c,
467      MiniBatchOperationInProgress<Mutation> miniBatchOp) throws IOException {}
468
469  /**
470   * This will be called for region operations where read lock is acquired in
471   * {@link Region#startRegionOperation()}.
472   * @param ctx
473   * @param operation The operation is about to be taken on the region
474   */
475  default void postStartRegionOperation(ObserverContext<RegionCoprocessorEnvironment> ctx,
476      Operation operation) throws IOException {}
477
478  /**
479   * Called after releasing read lock in {@link Region#closeRegionOperation()}.
480   * @param ctx
481   * @param operation
482   */
483  default void postCloseRegionOperation(ObserverContext<RegionCoprocessorEnvironment> ctx,
484      Operation operation) throws IOException {}
485
486  /**
487   * Called after the completion of batch put/delete and will be called even if the batch operation
488   * fails.
489   * <p>
490   * Note: Do not retain references to any Cells in Mutations beyond the life of this invocation.
491   * If need a Cell reference for later use, copy the cell and use that.
492   * @param ctx
493   * @param miniBatchOp
494   * @param success true if batch operation is successful otherwise false.
495   */
496  default void postBatchMutateIndispensably(ObserverContext<RegionCoprocessorEnvironment> ctx,
497      MiniBatchOperationInProgress<Mutation> miniBatchOp, boolean success) throws IOException {}
498
499  /**
500   * Called before checkAndPut.
501   * <p>
502   * Call CoprocessorEnvironment#bypass to skip default actions.
503   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
504   * <p>
505   * Note: Do not retain references to any Cells in 'put' beyond the life of this invocation.
506   * If need a Cell reference for later use, copy the cell and use that.
507   * @param c the environment provided by the region server
508   * @param row row to check
509   * @param family column family
510   * @param qualifier column qualifier
511   * @param op the comparison operation
512   * @param comparator the comparator
513   * @param put data to put if check succeeds
514   * @param result
515   * @return the return value to return to client if bypassing default
516   * processing
517   */
518  default boolean preCheckAndPut(ObserverContext<RegionCoprocessorEnvironment> c, byte[] row,
519      byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator, Put put,
520      boolean result) throws IOException {
521    return result;
522  }
523
524  /**
525   * Called before checkAndPut but after acquiring rowlock.
526   * <p>
527   * <b>Note:</b> Caution to be taken for not doing any long time operation in this hook.
528   * Row will be locked for longer time. Trying to acquire lock on another row, within this,
529   * can lead to potential deadlock.
530   * <p>
531   * Call CoprocessorEnvironment#bypass to skip default actions.
532   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
533   * <p>
534   * Note: Do not retain references to any Cells in 'put' beyond the life of this invocation.
535   * If need a Cell reference for later use, copy the cell and use that.
536   * @param c the environment provided by the region server
537   * @param row row to check
538   * @param family column family
539   * @param qualifier column qualifier
540   * @param op the comparison operation
541   * @param comparator the comparator
542   * @param put data to put if check succeeds
543   * @param result
544   * @return the return value to return to client if bypassing default
545   * processing
546   */
547  default boolean preCheckAndPutAfterRowLock(ObserverContext<RegionCoprocessorEnvironment> c,
548      byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
549      ByteArrayComparable comparator, Put put, boolean result) throws IOException {
550    return result;
551  }
552
553  /**
554   * Called after checkAndPut
555   * <p>
556   * Note: Do not retain references to any Cells in 'put' beyond the life of this invocation.
557   * If need a Cell reference for later use, copy the cell and use that.
558   * @param c the environment provided by the region server
559   * @param row row to check
560   * @param family column family
561   * @param qualifier column qualifier
562   * @param op the comparison operation
563   * @param comparator the comparator
564   * @param put data to put if check succeeds
565   * @param result from the checkAndPut
566   * @return the possibly transformed return value to return to client
567   */
568  default boolean postCheckAndPut(ObserverContext<RegionCoprocessorEnvironment> c, byte[] row,
569      byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator, Put put,
570      boolean result) throws IOException {
571    return result;
572  }
573
574  /**
575   * Called before checkAndDelete.
576   * <p>
577   * Call CoprocessorEnvironment#bypass to skip default actions.
578   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
579   * <p>
580   * Note: Do not retain references to any Cells in 'delete' beyond the life of this invocation.
581   * If need a Cell reference for later use, copy the cell and use that.
582   * @param c the environment provided by the region server
583   * @param row row to check
584   * @param family column family
585   * @param qualifier column qualifier
586   * @param op the comparison operation
587   * @param comparator the comparator
588   * @param delete delete to commit if check succeeds
589   * @param result
590   * @return the value to return to client if bypassing default processing
591   */
592  default boolean preCheckAndDelete(ObserverContext<RegionCoprocessorEnvironment> c, byte[] row,
593      byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator,
594      Delete delete, boolean result) throws IOException {
595    return result;
596  }
597
598  /**
599   * Called before checkAndDelete but after acquiring rowock.
600   * <p>
601   * <b>Note:</b> Caution to be taken for not doing any long time operation in this hook.
602   * Row will be locked for longer time. Trying to acquire lock on another row, within this,
603   * can lead to potential deadlock.
604   * <p>
605   * Call CoprocessorEnvironment#bypass to skip default actions.
606   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
607   * <p>
608   * Note: Do not retain references to any Cells in 'delete' beyond the life of this invocation.
609   * If need a Cell reference for later use, copy the cell and use that.
610   * @param c the environment provided by the region server
611   * @param row row to check
612   * @param family column family
613   * @param qualifier column qualifier
614   * @param op the comparison operation
615   * @param comparator the comparator
616   * @param delete delete to commit if check succeeds
617   * @param result
618   * @return the value to return to client if bypassing default processing
619   */
620  default boolean preCheckAndDeleteAfterRowLock(ObserverContext<RegionCoprocessorEnvironment> c,
621      byte[] row, byte[] family, byte[] qualifier, CompareOperator op,
622      ByteArrayComparable comparator, Delete delete, boolean result) throws IOException {
623    return result;
624  }
625
626  /**
627   * Called after checkAndDelete
628   * <p>
629   * Note: Do not retain references to any Cells in 'delete' beyond the life of this invocation.
630   * If need a Cell reference for later use, copy the cell and use that.
631   * @param c the environment provided by the region server
632   * @param row row to check
633   * @param family column family
634   * @param qualifier column qualifier
635   * @param op the comparison operation
636   * @param comparator the comparator
637   * @param delete delete to commit if check succeeds
638   * @param result from the CheckAndDelete
639   * @return the possibly transformed returned value to return to client
640   */
641  default boolean postCheckAndDelete(ObserverContext<RegionCoprocessorEnvironment> c, byte[] row,
642      byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator,
643      Delete delete, boolean result) throws IOException {
644    return result;
645  }
646
647  /**
648   * Called before Append.
649   * <p>
650   * Call CoprocessorEnvironment#bypass to skip default actions.
651   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
652   * <p>
653   * Note: Do not retain references to any Cells in 'append' beyond the life of this invocation.
654   * If need a Cell reference for later use, copy the cell and use that.
655   * @param c the environment provided by the region server
656   * @param append Append object
657   * @return result to return to the client if bypassing default processing
658   */
659  default Result preAppend(ObserverContext<RegionCoprocessorEnvironment> c, Append append)
660      throws IOException {
661    return null;
662  }
663
664  /**
665   * Called before Append but after acquiring rowlock.
666   * <p>
667   * <b>Note:</b> Caution to be taken for not doing any long time operation in this hook.
668   * Row will be locked for longer time. Trying to acquire lock on another row, within this,
669   * can lead to potential deadlock.
670   * <p>
671   * Call CoprocessorEnvironment#bypass to skip default actions.
672   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
673   * <p>
674   * Note: Do not retain references to any Cells in 'append' beyond the life of this invocation.
675   * If need a Cell reference for later use, copy the cell and use that.
676   * @param c the environment provided by the region server
677   * @param append Append object
678   * @return result to return to the client if bypassing default processing
679   */
680  default Result preAppendAfterRowLock(ObserverContext<RegionCoprocessorEnvironment> c,
681      Append append) throws IOException {
682    return null;
683  }
684
685  /**
686   * Called after Append
687   * <p>
688   * Note: Do not retain references to any Cells in 'append' beyond the life of this invocation.
689   * If need a Cell reference for later use, copy the cell and use that.
690   * @param c the environment provided by the region server
691   * @param append Append object
692   * @param result the result returned by increment
693   * @return the result to return to the client
694   */
695  default Result postAppend(ObserverContext<RegionCoprocessorEnvironment> c, Append append,
696      Result result) throws IOException {
697    return result;
698  }
699
700  /**
701   * Called before Increment.
702   * <p>
703   * Call CoprocessorEnvironment#bypass to skip default actions.
704   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
705   * <p>
706   * Note: Do not retain references to any Cells in 'increment' beyond the life of this invocation.
707   * If need a Cell reference for later use, copy the cell and use that.
708   * @param c the environment provided by the region server
709   * @param increment increment object
710   * @return result to return to the client if bypassing default processing
711   */
712  default Result preIncrement(ObserverContext<RegionCoprocessorEnvironment> c, Increment increment)
713      throws IOException {
714    return null;
715  }
716
717  /**
718   * Called before Increment but after acquiring rowlock.
719   * <p>
720   * <b>Note:</b> Caution to be taken for not doing any long time operation in this hook.
721   * Row will be locked for longer time. Trying to acquire lock on another row, within this,
722   * can lead to potential deadlock.
723   * <p>
724   * Call CoprocessorEnvironment#bypass to skip default actions.
725   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
726   * <p>
727   * Note: Do not retain references to any Cells in 'increment' beyond the life of this invocation.
728   * If need a Cell reference for later use, copy the cell and use that.
729   *
730   * @param c
731   *          the environment provided by the region server
732   * @param increment
733   *          increment object
734   * @return result to return to the client if bypassing default processing
735   *           if an error occurred on the coprocessor
736   */
737  default Result preIncrementAfterRowLock(ObserverContext<RegionCoprocessorEnvironment> c,
738      Increment increment) throws IOException {
739    return null;
740  }
741
742  /**
743   * Called after increment
744   * <p>
745   * Note: Do not retain references to any Cells in 'increment' beyond the life of this invocation.
746   * If need a Cell reference for later use, copy the cell and use that.
747   * @param c the environment provided by the region server
748   * @param increment increment object
749   * @param result the result returned by increment
750   * @return the result to return to the client
751   */
752  default Result postIncrement(ObserverContext<RegionCoprocessorEnvironment> c, Increment increment,
753      Result result) throws IOException {
754    return result;
755  }
756
757  /**
758   * Called before the client opens a new scanner.
759   * <p>
760   * Note: Do not retain references to any Cells returned by scanner, beyond the life of this
761   * invocation. If need a Cell reference for later use, copy the cell and use that.
762   * @param c the environment provided by the region server
763   * @param scan the Scan specification
764   */
765  default void preScannerOpen(ObserverContext<RegionCoprocessorEnvironment> c, Scan scan)
766      throws IOException {
767  }
768
769  /**
770   * Called after the client opens a new scanner.
771   * <p>
772   * Note: Do not retain references to any Cells returned by scanner, beyond the life of this
773   * invocation. If need a Cell reference for later use, copy the cell and use that.
774   * @param c the environment provided by the region server
775   * @param scan the Scan specification
776   * @param s if not null, the base scanner
777   * @return the scanner instance to use
778   */
779  default RegionScanner postScannerOpen(ObserverContext<RegionCoprocessorEnvironment> c, Scan scan,
780      RegionScanner s) throws IOException {
781    return s;
782  }
783
784  /**
785   * Called before the client asks for the next row on a scanner.
786   * <p>
787   * Call CoprocessorEnvironment#bypass to skip default actions.
788   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
789   * <p>
790   * Note: Do not retain references to any Cells returned by scanner, beyond the life of this
791   * invocation. If need a Cell reference for later use, copy the cell and use that.
792   * @param c the environment provided by the region server
793   * @param s the scanner
794   * @param result The result to return to the client if default processing
795   * is bypassed. Can be modified. Will not be returned if default processing
796   * is not bypassed.
797   * @param limit the maximum number of results to return
798   * @param hasNext the 'has more' indication
799   * @return 'has more' indication that should be sent to client
800   */
801  default boolean preScannerNext(ObserverContext<RegionCoprocessorEnvironment> c, InternalScanner s,
802      List<Result> result, int limit, boolean hasNext) throws IOException {
803    return hasNext;
804  }
805
806  /**
807   * Called after the client asks for the next row on a scanner.
808   * <p>
809   * Note: Do not retain references to any Cells returned by scanner, beyond the life of this
810   * invocation. If need a Cell reference for later use, copy the cell and use that.
811   * @param c the environment provided by the region server
812   * @param s the scanner
813   * @param result the result to return to the client, can be modified
814   * @param limit the maximum number of results to return
815   * @param hasNext the 'has more' indication
816   * @return 'has more' indication that should be sent to client
817   */
818  default boolean postScannerNext(ObserverContext<RegionCoprocessorEnvironment> c,
819      InternalScanner s, List<Result> result, int limit, boolean hasNext) throws IOException {
820    return hasNext;
821  }
822
823  /**
824   * This will be called by the scan flow when the current scanned row is being filtered out by the
825   * filter. The filter may be filtering out the row via any of the below scenarios
826   * <ol>
827   * <li>
828   * <code>boolean filterRowKey(byte [] buffer, int offset, int length)</code> returning true</li>
829   * <li>
830   * <code>boolean filterRow()</code> returning true</li>
831   * <li>
832   * <code>default void filterRow(List&lt;KeyValue&gt; kvs)</code> removing all the kvs from
833   * the passed List</li>
834   * </ol>
835   * <p>
836   * Note: Do not retain references to any Cells returned by scanner, beyond the life of this
837   * invocation. If need a Cell reference for later use, copy the cell and use that.
838   * @param c the environment provided by the region server
839   * @param s the scanner
840   * @param curRowCell The cell in the current row which got filtered out
841   * @param hasMore the 'has more' indication
842   * @return whether more rows are available for the scanner or not
843   */
844  default boolean postScannerFilterRow(ObserverContext<RegionCoprocessorEnvironment> c,
845      InternalScanner s, Cell curRowCell, boolean hasMore) throws IOException {
846    return hasMore;
847  }
848
849  /**
850   * Called before the client closes a scanner.
851   * <p>
852   * Call CoprocessorEnvironment#bypass to skip default actions.
853   * If 'bypass' is set, we skip out on calling any subsequent chained coprocessors.
854   * @param c the environment provided by the region server
855   * @param s the scanner
856   */
857  default void preScannerClose(ObserverContext<RegionCoprocessorEnvironment> c, InternalScanner s)
858      throws IOException {}
859
860  /**
861   * Called after the client closes a scanner.
862   * @param ctx the environment provided by the region server
863   * @param s the scanner
864   */
865  default void postScannerClose(ObserverContext<RegionCoprocessorEnvironment> ctx,
866      InternalScanner s) throws IOException {}
867
868  /**
869   * Called before a store opens a new scanner.
870   * <p>
871   * This hook is called when a "user" scanner is opened. Use {@code preFlushScannerOpen} and
872   * {@code preCompactScannerOpen} to inject flush/compaction.
873   * <p>
874   * Notice that, this method is used to change the inherent max versions and TTL for a Store. For
875   * example, you can change the max versions option for a {@link Scan} object to 10 in
876   * {@code preScannerOpen}, but if the max versions config on the Store is 1, then you still can
877   * only read 1 version. You need also to inject here to change the max versions to 10 if you want
878   * to get more versions.
879   * @param ctx the environment provided by the region server
880   * @param store the store which we want to get scanner from
881   * @param options used to change max versions and TTL for the scanner being opened
882   * @see #preFlushScannerOpen(ObserverContext, Store, ScanOptions, FlushLifeCycleTracker)
883   * @see #preCompactScannerOpen(ObserverContext, Store, ScanType, ScanOptions,
884   *      CompactionLifeCycleTracker, CompactionRequest)
885   */
886  default void preStoreScannerOpen(ObserverContext<RegionCoprocessorEnvironment> ctx, Store store,
887      ScanOptions options) throws IOException {}
888
889  /**
890   * Called before replaying WALs for this region.
891   * Calling {@link org.apache.hadoop.hbase.coprocessor.ObserverContext#bypass()} has no
892   * effect in this hook.
893   * @param ctx the environment provided by the region server
894   * @param info the RegionInfo for this region
895   * @param edits the file of recovered edits
896   */
897  // todo: what about these?
898  default void preReplayWALs(ObserverContext<? extends RegionCoprocessorEnvironment> ctx,
899    RegionInfo info, Path edits) throws IOException {}
900
901  /**
902   * Called after replaying WALs for this region.
903   * @param ctx the environment provided by the region server
904   * @param info the RegionInfo for this region
905   * @param edits the file of recovered edits
906   */
907  default void postReplayWALs(ObserverContext<? extends RegionCoprocessorEnvironment> ctx,
908    RegionInfo info, Path edits) throws IOException {}
909
910  /**
911   * Called before a {@link WALEdit}
912   * replayed for this region.
913   * @param ctx the environment provided by the region server
914   */
915  default void preWALRestore(ObserverContext<? extends RegionCoprocessorEnvironment> ctx,
916    RegionInfo info, WALKey logKey, WALEdit logEdit) throws IOException {}
917
918  /**
919   * Called after a {@link WALEdit}
920   * replayed for this region.
921   * @param ctx the environment provided by the region server
922   */
923  default void postWALRestore(ObserverContext<? extends RegionCoprocessorEnvironment> ctx,
924    RegionInfo info, WALKey logKey, WALEdit logEdit) throws IOException {}
925
926  /**
927   * Called before bulkLoadHFile. Users can create a StoreFile instance to
928   * access the contents of a HFile.
929   *
930   * @param ctx the environment provided by the region server
931   * @param familyPaths pairs of { CF, HFile path } submitted for bulk load. Adding
932   * or removing from this list will add or remove HFiles to be bulk loaded.
933   */
934  default void preBulkLoadHFile(ObserverContext<RegionCoprocessorEnvironment> ctx,
935    List<Pair<byte[], String>> familyPaths) throws IOException {}
936
937  /**
938   * Called before moving bulk loaded hfile to region directory.
939   *
940   * @param ctx the environment provided by the region server
941   * @param family column family
942   * @param pairs List of pairs of { HFile location in staging dir, HFile path in region dir }
943   * Each pair are for the same hfile.
944   */
945  default void preCommitStoreFile(ObserverContext<RegionCoprocessorEnvironment> ctx, byte[] family,
946      List<Pair<Path, Path>> pairs) throws IOException {}
947
948  /**
949   * Called after moving bulk loaded hfile to region directory.
950   *
951   * @param ctx the environment provided by the region server
952   * @param family column family
953   * @param srcPath Path to file before the move
954   * @param dstPath Path to file after the move
955   */
956  default void postCommitStoreFile(ObserverContext<RegionCoprocessorEnvironment> ctx, byte[] family,
957      Path srcPath, Path dstPath) throws IOException {}
958
959  /**
960   * Called after bulkLoadHFile.
961   *
962   * @param ctx the environment provided by the region server
963   * @param stagingFamilyPaths pairs of { CF, HFile path } submitted for bulk load
964   * @param finalPaths Map of CF to List of file paths for the loaded files
965   *   if the Map is not null, the bulkLoad was successful. Otherwise the bulk load failed.
966   *   bulkload is done by the time this hook is called.
967   */
968  default void postBulkLoadHFile(ObserverContext<RegionCoprocessorEnvironment> ctx,
969      List<Pair<byte[], String>> stagingFamilyPaths, Map<byte[], List<Path>> finalPaths)
970          throws IOException {
971  }
972
973  /**
974   * Called before creation of Reader for a store file.
975   * Calling {@link org.apache.hadoop.hbase.coprocessor.ObserverContext#bypass()} has no
976   * effect in this hook.
977   *
978   * @param ctx the environment provided by the region server
979   * @param fs fileystem to read from
980   * @param p path to the file
981   * @param in {@link FSDataInputStreamWrapper}
982   * @param size Full size of the file
983   * @param cacheConf
984   * @param r original reference file. This will be not null only when reading a split file.
985   * @param reader the base reader, if not {@code null}, from previous RegionObserver in the chain
986   * @return a Reader instance to use instead of the base reader if overriding
987   * default behavior, null otherwise
988   * @deprecated For Phoenix only, StoreFileReader is not a stable interface.
989   */
990  @Deprecated
991  // Passing InterfaceAudience.Private args FSDataInputStreamWrapper, CacheConfig and Reference.
992  // This is fine as the hook is deprecated any way.
993  default StoreFileReader preStoreFileReaderOpen(ObserverContext<RegionCoprocessorEnvironment> ctx,
994      FileSystem fs, Path p, FSDataInputStreamWrapper in, long size, CacheConfig cacheConf,
995      Reference r, StoreFileReader reader) throws IOException {
996    return reader;
997  }
998
999  /**
1000   * Called after the creation of Reader for a store file.
1001   *
1002   * @param ctx the environment provided by the region server
1003   * @param fs fileystem to read from
1004   * @param p path to the file
1005   * @param in {@link FSDataInputStreamWrapper}
1006   * @param size Full size of the file
1007   * @param cacheConf
1008   * @param r original reference file. This will be not null only when reading a split file.
1009   * @param reader the base reader instance
1010   * @return The reader to use
1011   * @deprecated For Phoenix only, StoreFileReader is not a stable interface.
1012   */
1013  @Deprecated
1014  // Passing InterfaceAudience.Private args FSDataInputStreamWrapper, CacheConfig and Reference.
1015  // This is fine as the hook is deprecated any way.
1016  default StoreFileReader postStoreFileReaderOpen(ObserverContext<RegionCoprocessorEnvironment> ctx,
1017      FileSystem fs, Path p, FSDataInputStreamWrapper in, long size, CacheConfig cacheConf,
1018      Reference r, StoreFileReader reader) throws IOException {
1019    return reader;
1020  }
1021
1022  /**
1023   * Called after a new cell has been created during an increment operation, but before
1024   * it is committed to the WAL or memstore.
1025   * Calling {@link org.apache.hadoop.hbase.coprocessor.ObserverContext#bypass()} has no
1026   * effect in this hook.
1027   * @param ctx the environment provided by the region server
1028   * @param opType the operation type
1029   * @param mutation the current mutation
1030   * @param oldCell old cell containing previous value
1031   * @param newCell the new cell containing the computed value
1032   * @return the new cell, possibly changed
1033   * @deprecated since 2.2.0 and will be removedin 4.0.0. Use
1034   *   {@link #postIncrementBeforeWAL(ObserverContext, Mutation, List)} or
1035   *   {@link #postAppendBeforeWAL(ObserverContext, Mutation, List)} instead.
1036   * @see #postIncrementBeforeWAL(ObserverContext, Mutation, List)
1037   * @see #postAppendBeforeWAL(ObserverContext, Mutation, List)
1038   * @see <a href="https://issues.apache.org/jira/browse/HBASE-21643">HBASE-21643</a>
1039   */
1040  @Deprecated
1041  default Cell postMutationBeforeWAL(ObserverContext<RegionCoprocessorEnvironment> ctx,
1042      MutationType opType, Mutation mutation, Cell oldCell, Cell newCell) throws IOException {
1043    return newCell;
1044  }
1045
1046  /**
1047   * Called after a list of new cells has been created during an increment operation, but before
1048   * they are committed to the WAL or memstore.
1049   *
1050   * @param ctx       the environment provided by the region server
1051   * @param mutation  the current mutation
1052   * @param cellPairs a list of cell pair. The first cell is old cell which may be null.
1053   *                  And the second cell is the new cell.
1054   * @return a list of cell pair, possibly changed.
1055   */
1056  default List<Pair<Cell, Cell>> postIncrementBeforeWAL(
1057      ObserverContext<RegionCoprocessorEnvironment> ctx, Mutation mutation,
1058      List<Pair<Cell, Cell>> cellPairs) throws IOException {
1059    List<Pair<Cell, Cell>> resultPairs = new ArrayList<>(cellPairs.size());
1060    for (Pair<Cell, Cell> pair : cellPairs) {
1061      resultPairs.add(new Pair<>(pair.getFirst(),
1062          postMutationBeforeWAL(ctx, MutationType.INCREMENT, mutation, pair.getFirst(),
1063              pair.getSecond())));
1064    }
1065    return resultPairs;
1066  }
1067
1068  /**
1069   * Called after a list of new cells has been created during an append operation, but before
1070   * they are committed to the WAL or memstore.
1071   *
1072   * @param ctx       the environment provided by the region server
1073   * @param mutation  the current mutation
1074   * @param cellPairs a list of cell pair. The first cell is old cell which may be null.
1075   *                  And the second cell is the new cell.
1076   * @return a list of cell pair, possibly changed.
1077   */
1078  default List<Pair<Cell, Cell>> postAppendBeforeWAL(
1079      ObserverContext<RegionCoprocessorEnvironment> ctx, Mutation mutation,
1080      List<Pair<Cell, Cell>> cellPairs) throws IOException {
1081    List<Pair<Cell, Cell>> resultPairs = new ArrayList<>(cellPairs.size());
1082    for (Pair<Cell, Cell> pair : cellPairs) {
1083      resultPairs.add(new Pair<>(pair.getFirst(),
1084          postMutationBeforeWAL(ctx, MutationType.INCREMENT, mutation, pair.getFirst(),
1085              pair.getSecond())));
1086    }
1087    return resultPairs;
1088  }
1089
1090  /**
1091   * Called after the ScanQueryMatcher creates ScanDeleteTracker. Implementing
1092   * this hook would help in creating customised DeleteTracker and returning
1093   * the newly created DeleteTracker
1094   * <p>
1095   * Warn: This is used by internal coprocessors. Should not be implemented by user coprocessors
1096   * @param ctx the environment provided by the region server
1097   * @param delTracker the deleteTracker that is created by the QueryMatcher
1098   * @return the Delete Tracker
1099   * @deprecated Since 2.0 with out any replacement and will be removed in 3.0
1100   */
1101  @Deprecated
1102  default DeleteTracker postInstantiateDeleteTracker(
1103      ObserverContext<RegionCoprocessorEnvironment> ctx, DeleteTracker delTracker)
1104      throws IOException {
1105    return delTracker;
1106  }
1107}