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.executor;
019
020import io.opentelemetry.api.trace.Span;
021import io.opentelemetry.context.Context;
022import io.opentelemetry.context.Scope;
023import java.io.IOException;
024import java.util.concurrent.atomic.AtomicLong;
025import org.apache.hadoop.hbase.Server;
026import org.apache.hadoop.hbase.trace.TraceUtil;
027import org.apache.yetus.audience.InterfaceAudience;
028import org.slf4j.Logger;
029import org.slf4j.LoggerFactory;
030import org.slf4j.MDC;
031
032/**
033 * Abstract base class for all HBase event handlers. Subclasses should implement the
034 * {@link #process()} and {@link #prepare()} methods. Subclasses should also do all necessary checks
035 * up in their prepare() if possible -- check table exists, is disabled, etc. -- so they fail fast
036 * rather than later when process is running. Do it this way because process be invoked directly but
037 * event handlers are also run in an executor context -- i.e. asynchronously -- and in this case,
038 * exceptions thrown at process time will not be seen by the invoker, not till we implement a
039 * call-back mechanism so the client can pick them up later.
040 * <p>
041 * Event handlers have an {@link EventType}. {@link EventType} is a list of ALL handler event types.
042 * We need to keep a full list in one place -- and as enums is a good shorthand for an
043 * implemenations -- because event handlers can be passed to executors when they are to be run
044 * asynchronously. The hbase executor, see ExecutorService, has a switch for passing event type to
045 * executor.
046 * <p>
047 * @see ExecutorService
048 */
049@InterfaceAudience.Private
050public abstract class EventHandler implements Runnable, Comparable<EventHandler> {
051  private static final Logger LOG = LoggerFactory.getLogger(EventHandler.class);
052
053  // type of event this object represents
054  protected EventType eventType;
055
056  protected Server server;
057
058  // sequence id generator for default FIFO ordering of events
059  protected static final AtomicLong seqids = new AtomicLong(0);
060
061  // sequence id for this event
062  private final long seqid;
063
064  // Time to wait for events to happen, should be kept short
065  protected int waitingTimeForEvents;
066
067  private final Span parent;
068
069  /**
070   * Default base class constructor.
071   */
072  public EventHandler(Server server, EventType eventType) {
073    this.parent = Span.current();
074    this.server = server;
075    this.eventType = eventType;
076    seqid = seqids.incrementAndGet();
077    if (server != null) {
078      this.waitingTimeForEvents =
079        server.getConfiguration().getInt("hbase.master.event.waiting.time", 1000);
080    }
081  }
082
083  /**
084   * Event handlers should do all the necessary checks in this method (rather than in the
085   * constructor, or in process()) so that the caller, which is mostly executed in the ipc context
086   * can fail fast. Process is executed async from the client ipc, so this method gives a quick
087   * chance to do some basic checks. Should be called after constructing the EventHandler, and
088   * before process().
089   * @return the instance of this class
090   * @throws Exception when something goes wrong
091   */
092  public EventHandler prepare() throws Exception {
093    return this;
094  }
095
096  @Override
097  public void run() {
098    Span span = TraceUtil.getGlobalTracer().spanBuilder(getClass().getSimpleName())
099      .setParent(Context.current().with(parent)).startSpan();
100    // assume that this is the top of an execution on a new or reused thread, that we're safe to
101    // blast any existing MDC state.
102    try (Scope scope = span.makeCurrent()) {
103      MDC.put("event_type", eventType.toString());
104      process();
105    } catch (Throwable t) {
106      handleException(t);
107    } finally {
108      span.end();
109      MDC.clear();
110    }
111  }
112
113  /**
114   * This method is the main processing loop to be implemented by the various subclasses.
115   */
116  public abstract void process() throws IOException;
117
118  /**
119   * Return the event type
120   * @return The event type.
121   */
122  public EventType getEventType() {
123    return this.eventType;
124  }
125
126  /**
127   * Get the priority level for this handler instance. This uses natural ordering so lower numbers
128   * are higher priority.
129   * <p>
130   * Lowest priority is Integer.MAX_VALUE. Highest priority is 0.
131   * <p>
132   * Subclasses should override this method to allow prioritizing handlers.
133   * <p>
134   * Handlers with the same priority are handled in FIFO order.
135   * <p>
136   * @return Integer.MAX_VALUE by default, override to set higher priorities
137   */
138  public int getPriority() {
139    return Integer.MAX_VALUE;
140  }
141
142  /** Returns This events' sequence id. */
143  public long getSeqid() {
144    return this.seqid;
145  }
146
147  /**
148   * Default prioritized runnable comparator which implements a FIFO ordering.
149   * <p>
150   * Subclasses should not override this. Instead, if they want to implement priority beyond FIFO,
151   * they should override {@link #getPriority()}.
152   */
153  @Override
154  public int compareTo(EventHandler o) {
155    if (o == null) {
156      return 1;
157    }
158    if (getPriority() != o.getPriority()) {
159      return (getPriority() < o.getPriority()) ? -1 : 1;
160    }
161    return (this.seqid < o.seqid) ? -1 : 1;
162  }
163
164  @Override
165  public String toString() {
166    return "Event #" + getSeqid() + " of type " + eventType + " (" + getInformativeName() + ")";
167  }
168
169  /**
170   * Event implementations should override thie class to provide an informative name about what
171   * event they are handling. For example, event-specific information such as which region or server
172   * is being processed should be included if possible.
173   */
174  public String getInformativeName() {
175    return this.getClass().toString();
176  }
177
178  /**
179   * Event exception handler, may be overridden
180   * @param t Throwable object
181   */
182  protected void handleException(Throwable t) {
183    String msg = "Caught throwable while processing event " + eventType;
184    LOG.error(msg, t);
185    if (server != null && (t instanceof Error || t instanceof RuntimeException)) {
186      server.abort(msg, t);
187    }
188  }
189}