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 */
018
019package org.apache.hadoop.hbase.coprocessor;
020
021import java.io.IOException;
022import java.util.ArrayList;
023import java.util.Collections;
024import java.util.Comparator;
025import java.util.HashSet;
026import java.util.List;
027import java.util.Optional;
028import java.util.Set;
029import java.util.TreeSet;
030import java.util.UUID;
031import java.util.concurrent.ConcurrentSkipListSet;
032import java.util.concurrent.atomic.AtomicInteger;
033import java.util.function.Function;
034
035import org.apache.yetus.audience.InterfaceAudience;
036import org.slf4j.Logger;
037import org.slf4j.LoggerFactory;
038import org.apache.hadoop.conf.Configuration;
039import org.apache.hadoop.fs.Path;
040import org.apache.hadoop.hbase.Abortable;
041import org.apache.hadoop.hbase.Coprocessor;
042import org.apache.hadoop.hbase.CoprocessorEnvironment;
043import org.apache.hadoop.hbase.DoNotRetryIOException;
044import org.apache.hadoop.hbase.HConstants;
045import org.apache.hadoop.hbase.ipc.RpcServer;
046import org.apache.hadoop.hbase.security.User;
047import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
048import org.apache.hadoop.hbase.util.CoprocessorClassLoader;
049import org.apache.hadoop.hbase.util.SortedList;
050
051/**
052 * Provides the common setup framework and runtime services for coprocessor
053 * invocation from HBase services.
054 * @param <C> type of specific coprocessor this host will handle
055 * @param <E> type of specific coprocessor environment this host requires.
056 * provides
057 */
058@InterfaceAudience.Private
059public abstract class CoprocessorHost<C extends Coprocessor, E extends CoprocessorEnvironment<C>> {
060  public static final String REGION_COPROCESSOR_CONF_KEY =
061      "hbase.coprocessor.region.classes";
062  public static final String REGIONSERVER_COPROCESSOR_CONF_KEY =
063      "hbase.coprocessor.regionserver.classes";
064  public static final String USER_REGION_COPROCESSOR_CONF_KEY =
065      "hbase.coprocessor.user.region.classes";
066  public static final String MASTER_COPROCESSOR_CONF_KEY =
067      "hbase.coprocessor.master.classes";
068  public static final String WAL_COPROCESSOR_CONF_KEY =
069    "hbase.coprocessor.wal.classes";
070  public static final String ABORT_ON_ERROR_KEY = "hbase.coprocessor.abortonerror";
071  public static final boolean DEFAULT_ABORT_ON_ERROR = true;
072  public static final String COPROCESSORS_ENABLED_CONF_KEY = "hbase.coprocessor.enabled";
073  public static final boolean DEFAULT_COPROCESSORS_ENABLED = true;
074  public static final String USER_COPROCESSORS_ENABLED_CONF_KEY =
075    "hbase.coprocessor.user.enabled";
076  public static final boolean DEFAULT_USER_COPROCESSORS_ENABLED = true;
077  public static final String SKIP_LOAD_DUPLICATE_TABLE_COPROCESSOR =
078      "hbase.skip.load.duplicate.table.coprocessor";
079  public static final boolean DEFAULT_SKIP_LOAD_DUPLICATE_TABLE_COPROCESSOR = false;
080
081  private static final Logger LOG = LoggerFactory.getLogger(CoprocessorHost.class);
082  protected Abortable abortable;
083  /** Ordered set of loaded coprocessors with lock */
084  protected final SortedList<E> coprocEnvironments =
085      new SortedList<>(new EnvironmentPriorityComparator());
086  protected Configuration conf;
087  // unique file prefix to use for local copies of jars when classloading
088  protected String pathPrefix;
089  protected AtomicInteger loadSequence = new AtomicInteger();
090
091  public CoprocessorHost(Abortable abortable) {
092    this.abortable = abortable;
093    this.pathPrefix = UUID.randomUUID().toString();
094  }
095
096  /**
097   * Not to be confused with the per-object _coprocessors_ (above),
098   * coprocessorNames is static and stores the set of all coprocessors ever
099   * loaded by any thread in this JVM. It is strictly additive: coprocessors are
100   * added to coprocessorNames, by checkAndLoadInstance() but are never removed, since
101   * the intention is to preserve a history of all loaded coprocessors for
102   * diagnosis in case of server crash (HBASE-4014).
103   */
104  private static Set<String> coprocessorNames =
105      Collections.synchronizedSet(new HashSet<String>());
106
107  public static Set<String> getLoadedCoprocessors() {
108    synchronized (coprocessorNames) {
109      return new HashSet(coprocessorNames);
110    }
111  }
112
113  /**
114   * Used to create a parameter to the HServerLoad constructor so that
115   * HServerLoad can provide information about the coprocessors loaded by this
116   * regionserver.
117   * (HBASE-4070: Improve region server metrics to report loaded coprocessors
118   * to master).
119   */
120  public Set<String> getCoprocessors() {
121    Set<String> returnValue = new TreeSet<>();
122    for (E e: coprocEnvironments) {
123      returnValue.add(e.getInstance().getClass().getSimpleName());
124    }
125    return returnValue;
126  }
127
128  /**
129   * Load system coprocessors once only. Read the class names from configuration.
130   * Called by constructor.
131   */
132  protected void loadSystemCoprocessors(Configuration conf, String confKey) {
133    boolean coprocessorsEnabled = conf.getBoolean(COPROCESSORS_ENABLED_CONF_KEY,
134      DEFAULT_COPROCESSORS_ENABLED);
135    if (!coprocessorsEnabled) {
136      return;
137    }
138
139    Class<?> implClass;
140
141    // load default coprocessors from configure file
142    String[] defaultCPClasses = conf.getStrings(confKey);
143    if (defaultCPClasses == null || defaultCPClasses.length == 0)
144      return;
145
146    int currentSystemPriority = Coprocessor.PRIORITY_SYSTEM;
147    for (String className : defaultCPClasses) {
148      String[] classNameAndPriority = className.split("\\|");
149      boolean hasPriorityOverride = false;
150      className = classNameAndPriority[0];
151      int overridePriority = Coprocessor.PRIORITY_SYSTEM;
152      if (classNameAndPriority.length > 1){
153        overridePriority = Integer.parseInt(classNameAndPriority[1]);
154        hasPriorityOverride = true;
155      }
156      className = className.trim();
157      if (findCoprocessor(className) != null) {
158        // If already loaded will just continue
159        LOG.warn("Attempted duplicate loading of " + className + "; skipped");
160        continue;
161      }
162      ClassLoader cl = this.getClass().getClassLoader();
163      Thread.currentThread().setContextClassLoader(cl);
164      try {
165        implClass = cl.loadClass(className);
166        int coprocPriority = hasPriorityOverride ? overridePriority : currentSystemPriority;
167        // Add coprocessors as we go to guard against case where a coprocessor is specified twice
168        // in the configuration
169        E env = checkAndLoadInstance(implClass, coprocPriority, conf);
170        if (env != null) {
171          this.coprocEnvironments.add(env);
172          LOG.info("System coprocessor {} loaded, priority={}.", className, coprocPriority);
173          if (!hasPriorityOverride) {
174            ++currentSystemPriority;
175          }
176        }
177      } catch (Throwable t) {
178        // We always abort if system coprocessors cannot be loaded
179        abortServer(className, t);
180      }
181    }
182  }
183
184  /**
185   * Load a coprocessor implementation into the host
186   * @param path path to implementation jar
187   * @param className the main class name
188   * @param priority chaining priority
189   * @param conf configuration for coprocessor
190   * @throws java.io.IOException Exception
191   */
192  public E load(Path path, String className, int priority,
193      Configuration conf) throws IOException {
194    String[] includedClassPrefixes = null;
195    if (conf.get(HConstants.CP_HTD_ATTR_INCLUSION_KEY) != null){
196      String prefixes = conf.get(HConstants.CP_HTD_ATTR_INCLUSION_KEY);
197      includedClassPrefixes = prefixes.split(";");
198    }
199    return load(path, className, priority, conf, includedClassPrefixes);
200  }
201
202  /**
203   * Load a coprocessor implementation into the host
204   * @param path path to implementation jar
205   * @param className the main class name
206   * @param priority chaining priority
207   * @param conf configuration for coprocessor
208   * @param includedClassPrefixes class name prefixes to include
209   * @throws java.io.IOException Exception
210   */
211  public E load(Path path, String className, int priority,
212      Configuration conf, String[] includedClassPrefixes) throws IOException {
213    Class<?> implClass;
214    LOG.debug("Loading coprocessor class " + className + " with path " +
215        path + " and priority " + priority);
216
217    boolean skipLoadDuplicateCoprocessor = conf.getBoolean(SKIP_LOAD_DUPLICATE_TABLE_COPROCESSOR,
218      DEFAULT_SKIP_LOAD_DUPLICATE_TABLE_COPROCESSOR);
219    if (skipLoadDuplicateCoprocessor && findCoprocessor(className) != null) {
220      // If already loaded will just continue
221      LOG.warn("Attempted duplicate loading of {}; skipped", className);
222      return null;
223    }
224
225    ClassLoader cl = null;
226    if (path == null) {
227      try {
228        implClass = getClass().getClassLoader().loadClass(className);
229      } catch (ClassNotFoundException e) {
230        throw new IOException("No jar path specified for " + className);
231      }
232    } else {
233      cl = CoprocessorClassLoader.getClassLoader(
234        path, getClass().getClassLoader(), pathPrefix, conf);
235      try {
236        implClass = ((CoprocessorClassLoader)cl).loadClass(className, includedClassPrefixes);
237      } catch (ClassNotFoundException e) {
238        throw new IOException("Cannot load external coprocessor class " + className, e);
239      }
240    }
241
242    //load custom code for coprocessor
243    Thread currentThread = Thread.currentThread();
244    ClassLoader hostClassLoader = currentThread.getContextClassLoader();
245    try{
246      // switch temporarily to the thread classloader for custom CP
247      currentThread.setContextClassLoader(cl);
248      E cpInstance = checkAndLoadInstance(implClass, priority, conf);
249      return cpInstance;
250    } finally {
251      // restore the fresh (host) classloader
252      currentThread.setContextClassLoader(hostClassLoader);
253    }
254  }
255
256  @VisibleForTesting
257  public void load(Class<? extends C> implClass, int priority, Configuration conf)
258      throws IOException {
259    E env = checkAndLoadInstance(implClass, priority, conf);
260    coprocEnvironments.add(env);
261  }
262
263  /**
264   * @param implClass Implementation class
265   * @param priority priority
266   * @param conf configuration
267   * @throws java.io.IOException Exception
268   */
269  public E checkAndLoadInstance(Class<?> implClass, int priority, Configuration conf)
270      throws IOException {
271    // create the instance
272    C impl;
273    try {
274      impl = checkAndGetInstance(implClass);
275      if (impl == null) {
276        LOG.error("Cannot load coprocessor " + implClass.getSimpleName());
277        return null;
278      }
279    } catch (InstantiationException|IllegalAccessException e) {
280      throw new IOException(e);
281    }
282    // create the environment
283    E env = createEnvironment(impl, priority, loadSequence.incrementAndGet(), conf);
284    assert env instanceof BaseEnvironment;
285    ((BaseEnvironment<C>) env).startup();
286    // HBASE-4014: maintain list of loaded coprocessors for later crash analysis
287    // if server (master or regionserver) aborts.
288    coprocessorNames.add(implClass.getName());
289    return env;
290  }
291
292  /**
293   * Called when a new Coprocessor class is loaded
294   */
295  public abstract E createEnvironment(C instance, int priority, int sequence, Configuration conf);
296
297  /**
298   * Called when a new Coprocessor class needs to be loaded. Checks if type of the given class
299   * is what the corresponding host implementation expects. If it is of correct type, returns an
300   * instance of the coprocessor to be loaded. If not, returns null.
301   * If an exception occurs when trying to create instance of a coprocessor, it's passed up and
302   * eventually results into server aborting.
303   */
304  public abstract C checkAndGetInstance(Class<?> implClass)
305      throws InstantiationException, IllegalAccessException;
306
307  public void shutdown(E e) {
308    assert e instanceof BaseEnvironment;
309    if (LOG.isDebugEnabled()) {
310      LOG.debug("Stop coprocessor " + e.getInstance().getClass().getName());
311    }
312    ((BaseEnvironment<C>) e).shutdown();
313  }
314
315  /**
316   * Find coprocessors by full class name or simple name.
317   */
318  public C findCoprocessor(String className) {
319    for (E env: coprocEnvironments) {
320      if (env.getInstance().getClass().getName().equals(className) ||
321          env.getInstance().getClass().getSimpleName().equals(className)) {
322        return env.getInstance();
323      }
324    }
325    return null;
326  }
327
328  @VisibleForTesting
329  public <T extends C> T findCoprocessor(Class<T> cls) {
330    for (E env: coprocEnvironments) {
331      if (cls.isAssignableFrom(env.getInstance().getClass())) {
332        return (T) env.getInstance();
333      }
334    }
335    return null;
336  }
337
338  /**
339   * Find list of coprocessors that extend/implement the given class/interface
340   * @param cls the class/interface to look for
341   * @return the list of coprocessors, or null if not found
342   */
343  public <T extends C> List<T> findCoprocessors(Class<T> cls) {
344    ArrayList<T> ret = new ArrayList<>();
345
346    for (E env: coprocEnvironments) {
347      C cp = env.getInstance();
348
349      if(cp != null) {
350        if (cls.isAssignableFrom(cp.getClass())) {
351          ret.add((T)cp);
352        }
353      }
354    }
355    return ret;
356  }
357
358  /**
359   * Find a coprocessor environment by class name
360   * @param className the class name
361   * @return the coprocessor, or null if not found
362   */
363  @VisibleForTesting
364  public E findCoprocessorEnvironment(String className) {
365    for (E env: coprocEnvironments) {
366      if (env.getInstance().getClass().getName().equals(className) ||
367          env.getInstance().getClass().getSimpleName().equals(className)) {
368        return env;
369      }
370    }
371    return null;
372  }
373
374  /**
375   * Retrieves the set of classloaders used to instantiate Coprocessor classes defined in external
376   * jar files.
377   * @return A set of ClassLoader instances
378   */
379  Set<ClassLoader> getExternalClassLoaders() {
380    Set<ClassLoader> externalClassLoaders = new HashSet<>();
381    final ClassLoader systemClassLoader = this.getClass().getClassLoader();
382    for (E env : coprocEnvironments) {
383      ClassLoader cl = env.getInstance().getClass().getClassLoader();
384      if (cl != systemClassLoader){
385        //do not include system classloader
386        externalClassLoaders.add(cl);
387      }
388    }
389    return externalClassLoaders;
390  }
391
392  /**
393   * Environment priority comparator.
394   * Coprocessors are chained in sorted order.
395   */
396  static class EnvironmentPriorityComparator implements Comparator<CoprocessorEnvironment> {
397    @Override
398    public int compare(final CoprocessorEnvironment env1,
399        final CoprocessorEnvironment env2) {
400      if (env1.getPriority() < env2.getPriority()) {
401        return -1;
402      } else if (env1.getPriority() > env2.getPriority()) {
403        return 1;
404      }
405      if (env1.getLoadSequence() < env2.getLoadSequence()) {
406        return -1;
407      } else if (env1.getLoadSequence() > env2.getLoadSequence()) {
408        return 1;
409      }
410      return 0;
411    }
412  }
413
414  protected void abortServer(final E environment, final Throwable e) {
415    abortServer(environment.getInstance().getClass().getName(), e);
416  }
417
418  protected void abortServer(final String coprocessorName, final Throwable e) {
419    String message = "The coprocessor " + coprocessorName + " threw " + e.toString();
420    LOG.error(message, e);
421    if (abortable != null) {
422      abortable.abort(message, e);
423    } else {
424      LOG.warn("No available Abortable, process was not aborted");
425    }
426  }
427
428  /**
429   * This is used by coprocessor hooks which are declared to throw IOException
430   * (or its subtypes). For such hooks, we should handle throwable objects
431   * depending on the Throwable's type. Those which are instances of
432   * IOException should be passed on to the client. This is in conformance with
433   * the HBase idiom regarding IOException: that it represents a circumstance
434   * that should be passed along to the client for its own handling. For
435   * example, a coprocessor that implements access controls would throw a
436   * subclass of IOException, such as AccessDeniedException, in its preGet()
437   * method to prevent an unauthorized client's performing a Get on a particular
438   * table.
439   * @param env Coprocessor Environment
440   * @param e Throwable object thrown by coprocessor.
441   * @exception IOException Exception
442   */
443  // Note to devs: Class comments of all observers ({@link MasterObserver}, {@link WALObserver},
444  // etc) mention this nuance of our exception handling so that coprocessor can throw appropriate
445  // exceptions depending on situation. If any changes are made to this logic, make sure to
446  // update all classes' comments.
447  protected void handleCoprocessorThrowable(final E env, final Throwable e) throws IOException {
448    if (e instanceof IOException) {
449      throw (IOException)e;
450    }
451    // If we got here, e is not an IOException. A loaded coprocessor has a
452    // fatal bug, and the server (master or regionserver) should remove the
453    // faulty coprocessor from its set of active coprocessors. Setting
454    // 'hbase.coprocessor.abortonerror' to true will cause abortServer(),
455    // which may be useful in development and testing environments where
456    // 'failing fast' for error analysis is desired.
457    if (env.getConfiguration().getBoolean(ABORT_ON_ERROR_KEY, DEFAULT_ABORT_ON_ERROR)) {
458      // server is configured to abort.
459      abortServer(env, e);
460    } else {
461      // If available, pull a table name out of the environment
462      if(env instanceof RegionCoprocessorEnvironment) {
463        String tableName = ((RegionCoprocessorEnvironment)env).getRegionInfo().getTable().getNameAsString();
464        LOG.error("Removing coprocessor '" + env.toString() + "' from table '"+ tableName + "'", e);
465      } else {
466        LOG.error("Removing coprocessor '" + env.toString() + "' from " +
467                "environment",e);
468      }
469
470      coprocEnvironments.remove(env);
471      try {
472        shutdown(env);
473      } catch (Exception x) {
474        LOG.error("Uncaught exception when shutting down coprocessor '"
475            + env.toString() + "'", x);
476      }
477      throw new DoNotRetryIOException("Coprocessor: '" + env.toString() +
478          "' threw: '" + e + "' and has been removed from the active " +
479          "coprocessor set.", e);
480    }
481  }
482
483  /**
484   * Used to limit legacy handling to once per Coprocessor class per classloader.
485   */
486  private static final Set<Class<? extends Coprocessor>> legacyWarning =
487      new ConcurrentSkipListSet<>(
488          new Comparator<Class<? extends Coprocessor>>() {
489            @Override
490            public int compare(Class<? extends Coprocessor> c1, Class<? extends Coprocessor> c2) {
491              if (c1.equals(c2)) {
492                return 0;
493              }
494              return c1.getName().compareTo(c2.getName());
495            }
496          });
497
498  /**
499   * Implementations defined function to get an observer of type {@code O} from a coprocessor of
500   * type {@code C}. Concrete implementations of CoprocessorHost define one getter for each
501   * observer they can handle. For e.g. RegionCoprocessorHost will use 3 getters, one for
502   * each of RegionObserver, EndpointObserver and BulkLoadObserver.
503   * These getters are used by {@code ObserverOperation} to get appropriate observer from the
504   * coprocessor.
505   */
506  @FunctionalInterface
507  public interface ObserverGetter<C, O> extends Function<C, Optional<O>> {}
508
509  private abstract class ObserverOperation<O> extends ObserverContextImpl<E> {
510    ObserverGetter<C, O> observerGetter;
511
512    ObserverOperation(ObserverGetter<C, O> observerGetter) {
513      this(observerGetter, null);
514    }
515
516    ObserverOperation(ObserverGetter<C, O> observerGetter, User user) {
517      this(observerGetter, user, false);
518    }
519
520    ObserverOperation(ObserverGetter<C, O> observerGetter, boolean bypassable) {
521      this(observerGetter, null, bypassable);
522    }
523
524    ObserverOperation(ObserverGetter<C, O> observerGetter, User user, boolean bypassable) {
525      super(user != null? user: RpcServer.getRequestUser().orElse(null), bypassable);
526      this.observerGetter = observerGetter;
527    }
528
529    abstract void callObserver() throws IOException;
530    protected void postEnvCall() {}
531  }
532
533  // Can't derive ObserverOperation from ObserverOperationWithResult (R = Void) because then all
534  // ObserverCaller implementations will have to have a return statement.
535  // O = observer, E = environment, C = coprocessor, R=result type
536  public abstract class ObserverOperationWithoutResult<O> extends ObserverOperation<O> {
537    protected abstract void call(O observer) throws IOException;
538
539    public ObserverOperationWithoutResult(ObserverGetter<C, O> observerGetter) {
540      super(observerGetter);
541    }
542
543    public ObserverOperationWithoutResult(ObserverGetter<C, O> observerGetter, User user) {
544      super(observerGetter, user);
545    }
546
547    public ObserverOperationWithoutResult(ObserverGetter<C, O> observerGetter, User user,
548        boolean bypassable) {
549      super(observerGetter, user, bypassable);
550    }
551
552    /**
553     * In case of coprocessors which have many kinds of observers (for eg, {@link RegionCoprocessor}
554     * has BulkLoadObserver, RegionObserver, etc), some implementations may not need all
555     * observers, in which case they will return null for that observer's getter.
556     * We simply ignore such cases.
557     */
558    @Override
559    void callObserver() throws IOException {
560      Optional<O> observer = observerGetter.apply(getEnvironment().getInstance());
561      if (observer.isPresent()) {
562        call(observer.get());
563      }
564    }
565  }
566
567  public abstract class ObserverOperationWithResult<O, R> extends ObserverOperation<O> {
568    protected abstract R call(O observer) throws IOException;
569
570    private R result;
571
572    public ObserverOperationWithResult(ObserverGetter<C, O> observerGetter, R result) {
573      this(observerGetter, result, false);
574    }
575
576    public ObserverOperationWithResult(ObserverGetter<C, O> observerGetter, R result,
577        boolean bypassable) {
578      this(observerGetter, result, null, bypassable);
579    }
580
581    public ObserverOperationWithResult(ObserverGetter<C, O> observerGetter, R result,
582        User user) {
583      this(observerGetter, result, user, false);
584    }
585
586    private ObserverOperationWithResult(ObserverGetter<C, O> observerGetter, R result, User user,
587        boolean bypassable) {
588      super(observerGetter, user, bypassable);
589      this.result = result;
590    }
591
592    protected R getResult() {
593      return this.result;
594    }
595
596    @Override
597    void callObserver() throws IOException {
598      Optional<O> observer = observerGetter.apply(getEnvironment().getInstance());
599      if (observer.isPresent()) {
600        result = call(observer.get());
601      }
602    }
603  }
604
605  //////////////////////////////////////////////////////////////////////////////////////////
606  // Functions to execute observer hooks and handle results (if any)
607  //////////////////////////////////////////////////////////////////////////////////////////
608
609  /**
610   * Do not call with an observerOperation that is null! Have the caller check.
611   */
612  protected <O, R> R execOperationWithResult(
613      final ObserverOperationWithResult<O, R> observerOperation) throws IOException {
614    boolean bypass = execOperation(observerOperation);
615    R result = observerOperation.getResult();
616    return bypass == observerOperation.isBypassable()? result: null;
617  }
618
619  /**
620   * @return True if we are to bypass (Can only be <code>true</code> if
621   * ObserverOperation#isBypassable().
622   */
623  protected <O> boolean execOperation(final ObserverOperation<O> observerOperation)
624      throws IOException {
625    boolean bypass = false;
626    if (observerOperation == null) {
627      return bypass;
628    }
629    List<E> envs = coprocEnvironments.get();
630    for (E env : envs) {
631      observerOperation.prepare(env);
632      Thread currentThread = Thread.currentThread();
633      ClassLoader cl = currentThread.getContextClassLoader();
634      try {
635        currentThread.setContextClassLoader(env.getClassLoader());
636        observerOperation.callObserver();
637      } catch (Throwable e) {
638        handleCoprocessorThrowable(env, e);
639      } finally {
640        currentThread.setContextClassLoader(cl);
641      }
642      // Internal to shouldBypass, it checks if obeserverOperation#isBypassable().
643      bypass |= observerOperation.shouldBypass();
644      observerOperation.postEnvCall();
645      if (bypass) {
646        // If CP says bypass, skip out w/o calling any following CPs; they might ruin our response.
647        // In hbase1, this used to be called 'complete'. In hbase2, we unite bypass and 'complete'.
648        break;
649      }
650    }
651    return bypass;
652  }
653
654  /**
655   * Coprocessor classes can be configured in any order, based on that priority is set and
656   * chained in a sorted order. Should be used preStop*() hooks i.e. when master/regionserver is
657   * going down. This function first calls coprocessor methods (using ObserverOperation.call())
658   * and then shutdowns the environment in postEnvCall(). <br>
659   * Need to execute all coprocessor methods first then postEnvCall(), otherwise some coprocessors
660   * may remain shutdown if any exception occurs during next coprocessor execution which prevent
661   * master/regionserver stop or cluster shutdown. (Refer:
662   * <a href="https://issues.apache.org/jira/browse/HBASE-16663">HBASE-16663</a>
663   * @return true if bypaas coprocessor execution, false if not.
664   * @throws IOException
665   */
666  protected <O> boolean execShutdown(final ObserverOperation<O> observerOperation)
667      throws IOException {
668    if (observerOperation == null) return false;
669    boolean bypass = false;
670    List<E> envs = coprocEnvironments.get();
671    // Iterate the coprocessors and execute ObserverOperation's call()
672    for (E env : envs) {
673      observerOperation.prepare(env);
674      Thread currentThread = Thread.currentThread();
675      ClassLoader cl = currentThread.getContextClassLoader();
676      try {
677        currentThread.setContextClassLoader(env.getClassLoader());
678        observerOperation.callObserver();
679      } catch (Throwable e) {
680        handleCoprocessorThrowable(env, e);
681      } finally {
682        currentThread.setContextClassLoader(cl);
683      }
684      bypass |= observerOperation.shouldBypass();
685    }
686
687    // Iterate the coprocessors and execute ObserverOperation's postEnvCall()
688    for (E env : envs) {
689      observerOperation.prepare(env);
690      observerOperation.postEnvCall();
691    }
692    return bypass;
693  }
694}