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.procedure2.store;
019
020import java.io.IOException;
021
022import org.apache.hadoop.hbase.procedure2.Procedure;
023import org.apache.yetus.audience.InterfaceAudience;
024import org.apache.yetus.audience.InterfaceStability;
025
026/**
027 * The ProcedureStore is used by the executor to persist the state of each procedure execution. This
028 * allows to resume the execution of pending/in-progress procedures in case of machine failure or
029 * service shutdown.
030 * <p/>
031 * Notice that, the implementation must guarantee that the maxProcId when loading is the maximum one
032 * in the whole history, not only the current live procedures. This is very important as for
033 * executing remote procedures, we have some nonce checks at region server side to prevent executing
034 * non-idempotent operations more than once. If the procedure id could go back, then we may
035 * accidentally ignore some important operations such as region assign or unassign.<br/>
036 * This may lead to some garbages so we provide a {@link #cleanup()} method, the framework will call
037 * this method periodically and the store implementation could do some clean up works in this
038 * method.
039 */
040@InterfaceAudience.Private
041@InterfaceStability.Evolving
042public interface ProcedureStore {
043  /**
044   * Store listener interface.
045   * <p/>
046   * The main process should register a listener and respond to the store events.
047   */
048  public interface ProcedureStoreListener {
049
050    /**
051     * triggered when the store sync is completed.
052     */
053    default void postSync() {
054    }
055
056    /**
057     * triggered when the store is not able to write out data. the main process should abort.
058     */
059    default void abortProcess() {
060    }
061
062    /**
063     * Suggest that the upper layer should update the state of some procedures. Ignore this call
064     * will not effect correctness but performance.
065     * <p/>
066     * For a WAL based ProcedureStore implementation, if all the procedures stored in a WAL file
067     * have been deleted, or updated later in another WAL file, then we can delete the WAL file. If
068     * there are old procedures in a WAL file which are never deleted or updated, then we can not
069     * delete the WAL file and this will cause we hold lots of WAL file and slow down the master
070     * restarts. So here we introduce this method to tell the upper layer that please update the
071     * states of these procedures so that we can delete the old WAL file.
072     * @param procIds the id for the procedures
073     */
074    default void forceUpdate(long[] procIds) {
075    }
076  }
077
078  /**
079   * An Iterator over a collection of Procedure
080   */
081  public interface ProcedureIterator {
082    /**
083     * Reset the Iterator by seeking to the beginning of the list.
084     */
085    void reset();
086
087    /**
088     * Returns true if the iterator has more elements.
089     * (In other words, returns true if next() would return a Procedure
090     * rather than throwing an exception.)
091     * @return true if the iterator has more procedures
092     */
093    boolean hasNext();
094
095    /**
096     * Calling this method does not need to convert the protobuf message to the Procedure class, so
097     * if it returns true we can call {@link #skipNext()} to skip the procedure without
098     * deserializing. This could increase the performance.
099     * @return true if the iterator next element is a completed procedure.
100     */
101    boolean isNextFinished();
102
103    /**
104     * Skip the next procedure
105     * <p/>
106     * This method is used to skip the deserializing of the procedure to increase performance, as
107     * when calling next we need to convert the protobuf message to the Procedure class.
108     */
109    void skipNext();
110
111    /**
112     * Returns the next procedure in the iteration.
113     * @throws IOException if there was an error fetching/deserializing the procedure
114     * @return the next procedure in the iteration.
115     */
116    @SuppressWarnings("rawtypes")
117    Procedure next() throws IOException;
118  }
119
120  /**
121   * Interface passed to the ProcedureStore.load() method to handle the store-load events.
122   */
123  public interface ProcedureLoader {
124    /**
125     * Called by ProcedureStore.load() to notify about the maximum proc-id in the store.
126     * @param maxProcId the highest proc-id in the store
127     */
128    void setMaxProcId(long maxProcId);
129
130    /**
131     * Called by the ProcedureStore.load() every time a set of procedures are ready to be executed.
132     * The ProcedureIterator passed to the method, has the procedure sorted in replay-order.
133     * @param procIter iterator over the procedures ready to be added to the executor.
134     */
135    void load(ProcedureIterator procIter) throws IOException;
136
137    /**
138     * Called by the ProcedureStore.load() in case we have procedures not-ready to be added to
139     * the executor, which probably means they are corrupted since some information/link is missing.
140     * @param procIter iterator over the procedures not ready to be added to the executor, corrupted
141     */
142    void handleCorrupted(ProcedureIterator procIter) throws IOException;
143  }
144
145  /**
146   * Add the listener to the notification list.
147   * @param listener The AssignmentListener to register
148   */
149  void registerListener(ProcedureStoreListener listener);
150
151  /**
152   * Remove the listener from the notification list.
153   * @param listener The AssignmentListener to unregister
154   * @return true if the listner was in the list and it was removed, otherwise false.
155   */
156  boolean unregisterListener(ProcedureStoreListener listener);
157
158  /**
159   * Start/Open the procedure store
160   * @param numThreads number of threads to be used by the procedure store
161   */
162  void start(int numThreads) throws IOException;
163
164  /**
165   * Stop/Close the procedure store
166   * @param abort true if the stop is an abort
167   */
168  void stop(boolean abort);
169
170  /**
171   * @return true if the store is running, otherwise false.
172   */
173  boolean isRunning();
174
175  /**
176   * @return the number of threads/slots passed to start()
177   */
178  int getNumThreads();
179
180  /**
181   * Set the number of procedure running.
182   * This can be used, for example, by the store to know how long to wait before a sync.
183   * @return how many procedures are running (may not be same as <code>count</code>).
184   */
185  int setRunningProcedureCount(int count);
186
187  /**
188   * Acquire the lease for the procedure store.
189   * @deprecated since 2.3.0, will be removed in 4.0.0 along with
190   *             {@link org.apache.hadoop.hbase.procedure2.store.wal.WALProcedureStore}. As now we
191   *             will store the procedure data in a master local region, and master itself will deal
192   *             with the lease recovery of the region.
193   */
194  @Deprecated
195  void recoverLease() throws IOException;
196
197  /**
198   * Load the Procedures in the store.
199   * @param loader the ProcedureLoader that will handle the store-load events
200   */
201  void load(ProcedureLoader loader) throws IOException;
202
203  /**
204   * When a procedure is submitted to the executor insert(proc, null) will be called.
205   * 'proc' has a 'RUNNABLE' state and the initial information required to start up.
206   *
207   * When a procedure is executed and it returns children insert(proc, subprocs) will be called.
208   * 'proc' has a 'WAITING' state and an update state.
209   * 'subprocs' are the children in 'RUNNABLE' state with the initial information.
210   *
211   * @param proc the procedure to serialize and write to the store.
212   * @param subprocs the newly created child of the proc.
213   */
214  void insert(Procedure<?> proc, Procedure<?>[] subprocs);
215
216  /**
217   * Serialize a set of new procedures.
218   * These procedures are freshly submitted to the executor and each procedure
219   * has a 'RUNNABLE' state and the initial information required to start up.
220   *
221   * @param procs the procedures to serialize and write to the store.
222   */
223  void insert(Procedure<?>[] procs);
224
225  /**
226   * The specified procedure was executed,
227   * and the new state should be written to the store.
228   * @param proc the procedure to serialize and write to the store.
229   */
230  void update(Procedure<?> proc);
231
232  /**
233   * The specified procId was removed from the executor,
234   * due to completion, abort or failure.
235   * The store implementor should remove all the information about the specified procId.
236   * @param procId the ID of the procedure to remove.
237   */
238  void delete(long procId);
239
240  /**
241   * The parent procedure completed.
242   * Update the state and mark all the child deleted.
243   * @param parentProc the parent procedure to serialize and write to the store.
244   * @param subProcIds the IDs of the sub-procedure to remove.
245   */
246  void delete(Procedure<?> parentProc, long[] subProcIds);
247
248  /**
249   * The specified procIds were removed from the executor,
250   * due to completion, abort or failure.
251   * The store implementor should remove all the information about the specified procIds.
252   * @param procIds the IDs of the procedures to remove.
253   * @param offset the array offset from where to start to delete
254   * @param count the number of IDs to delete
255   */
256  void delete(long[] procIds, int offset, int count);
257
258  /**
259   * Will be called by the framework to give the store a chance to do some clean up works.
260   * <p/>
261   * Notice that this is for periodical clean up work, not for the clean up after close, if you want
262   * to close the store just call the {@link #stop(boolean)} method above.
263   */
264  default void cleanup() {
265  }
266}