View Javadoc

1   /**
2    * Licensed to the Apache Software Foundation (ASF) under one
3    * or more contributor license agreements.  See the NOTICE file
4    * distributed with this work for additional information
5    * regarding copyright ownership.  The ASF licenses this file
6    * to you under the Apache License, Version 2.0 (the
7    * "License"); you may not use this file except in compliance
8    * with the License.  You may obtain a copy of the License at
9    *
10   *     http://www.apache.org/licenses/LICENSE-2.0
11   *
12   * Unless required by applicable law or agreed to in writing, software
13   * distributed under the License is distributed on an "AS IS" BASIS,
14   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15   * See the License for the specific language governing permissions and
16   * limitations under the License.
17   */
18  
19  package org.apache.hadoop.hbase.procedure2;
20  
21  import java.util.ArrayList;
22  import java.util.List;
23  
24  import org.apache.commons.logging.Log;
25  import org.apache.commons.logging.LogFactory;
26  import org.apache.hadoop.hbase.classification.InterfaceAudience;
27  import org.apache.hadoop.hbase.classification.InterfaceStability;
28  import org.apache.hadoop.hbase.protobuf.generated.ProcedureProtos.ProcedureState;
29  
30  /**
31   * Internal state of the ProcedureExecutor that describes the state of a "Root Procedure".
32   * A "Root Procedure" is a Procedure without parent, each subprocedure will be
33   * added to the "Root Procedure" stack (or rollback-stack).
34   *
35   * RootProcedureState is used and managed only by the ProcedureExecutor.
36   *    Long rootProcId = getRootProcedureId(proc);
37   *    rollbackStack.get(rootProcId).acquire(proc)
38   *    rollbackStack.get(rootProcId).release(proc)
39   *    ...
40   */
41  @InterfaceAudience.Private
42  @InterfaceStability.Evolving
43  class RootProcedureState {
44    private static final Log LOG = LogFactory.getLog(RootProcedureState.class);
45  
46    private enum State {
47      RUNNING,         // The Procedure is running or ready to run
48      FAILED,          // The Procedure failed, waiting for the rollback executing
49      ROLLINGBACK,     // The Procedure failed and the execution was rolledback
50    }
51  
52    private ArrayList<Procedure> subprocedures = null;
53    private State state = State.RUNNING;
54    private int running = 0;
55  
56    public synchronized boolean isFailed() {
57      switch (state) {
58        case ROLLINGBACK:
59        case FAILED:
60          return true;
61        default:
62          break;
63      }
64      return false;
65    }
66  
67    public synchronized boolean isRollingback() {
68      return state == State.ROLLINGBACK;
69    }
70  
71    /**
72     * Called by the ProcedureExecutor to mark rollback execution
73     */
74    protected synchronized boolean setRollback() {
75      if (running == 0 && state == State.FAILED) {
76        state = State.ROLLINGBACK;
77        return true;
78      }
79      return false;
80    }
81  
82    /**
83     * Called by the ProcedureExecutor to mark rollback execution
84     */
85    protected synchronized void unsetRollback() {
86      assert state == State.ROLLINGBACK;
87      state = State.FAILED;
88    }
89  
90    protected synchronized List<Procedure> getSubprocedures() {
91      return subprocedures;
92    }
93  
94    protected synchronized RemoteProcedureException getException() {
95      if (subprocedures != null) {
96        for (Procedure proc: subprocedures) {
97          if (proc.hasException()) {
98            return proc.getException();
99          }
100       }
101     }
102     return null;
103   }
104 
105   /**
106    * Called by the ProcedureExecutor to mark the procedure step as running.
107    */
108   protected synchronized boolean acquire(final Procedure proc) {
109     if (state != State.RUNNING) return false;
110 
111     running++;
112     return true;
113   }
114 
115   /**
116    * Called by the ProcedureExecutor to mark the procedure step as finished.
117    */
118   protected synchronized void release(final Procedure proc) {
119     running--;
120   }
121 
122   protected synchronized void abort() {
123     if (state == State.RUNNING) {
124       state = State.FAILED;
125     }
126   }
127 
128   /**
129    * Called by the ProcedureExecutor after the procedure step is completed,
130    * to add the step to the rollback list (or procedure stack)
131    */
132   protected synchronized void addRollbackStep(final Procedure proc) {
133     if (proc.isFailed()) {
134       state = State.FAILED;
135     }
136     if (subprocedures == null) {
137       subprocedures = new ArrayList<Procedure>();
138     }
139     proc.addStackIndex(subprocedures.size());
140     subprocedures.add(proc);
141   }
142 
143   /**
144    * Called on store load by the ProcedureExecutor to load part of the stack.
145    *
146    * Each procedure has its own stack-positions. Which means we have to write
147    * to the store only the Procedure we executed, and nothing else.
148    * on load we recreate the full stack by aggregating each procedure stack-positions.
149    */
150   protected synchronized void loadStack(final Procedure proc) {
151     int[] stackIndexes = proc.getStackIndexes();
152     if (stackIndexes != null) {
153       if (subprocedures == null) {
154         subprocedures = new ArrayList<Procedure>();
155       }
156       int diff = (1 + stackIndexes[stackIndexes.length - 1]) - subprocedures.size();
157       if (diff > 0) {
158         subprocedures.ensureCapacity(1 + stackIndexes[stackIndexes.length - 1]);
159         while (diff-- > 0) subprocedures.add(null);
160       }
161       for (int i = 0; i < stackIndexes.length; ++i) {
162         subprocedures.set(stackIndexes[i], proc);
163       }
164     }
165     if (proc.getState() == ProcedureState.ROLLEDBACK) {
166       state = State.ROLLINGBACK;
167     } else if (proc.isFailed()) {
168       state = State.FAILED;
169     }
170   }
171 
172   /**
173    * Called on store load by the ProcedureExecutor to validate the procedure stack.
174    */
175   protected synchronized boolean isValid() {
176     if (subprocedures != null) {
177       for (Procedure proc: subprocedures) {
178         if (proc == null) {
179           return false;
180         }
181       }
182     }
183     return true;
184   }
185 }