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  package org.apache.hadoop.hbase.master;
19  
20  import java.util.Date;
21  import java.util.concurrent.atomic.AtomicLong;
22  
23  import org.apache.hadoop.hbase.classification.InterfaceAudience;
24  import org.apache.hadoop.hbase.classification.InterfaceStability;
25  import org.apache.hadoop.hbase.HRegionInfo;
26  import org.apache.hadoop.hbase.ServerName;
27  import org.apache.hadoop.hbase.protobuf.generated.ClusterStatusProtos;
28  
29  /**
30   * State of a Region while undergoing transitions.
31   * Region state cannot be modified except the stamp field.
32   * So it is almost immutable.
33   */
34  @InterfaceAudience.Private
35  public class RegionState {
36  
37    @InterfaceAudience.Private
38    @InterfaceStability.Evolving
39    public enum State {
40      OFFLINE,        // region is in an offline state
41      PENDING_OPEN,   // sent rpc to server to open but has not begun
42      OPENING,        // server has begun to open but not yet done
43      OPEN,           // server opened region and updated meta
44      PENDING_CLOSE,  // sent rpc to server to close but has not begun
45      CLOSING,        // server has begun to close but not yet done
46      CLOSED,         // server closed region and updated meta
47      SPLITTING,      // server started split of a region
48      SPLIT,          // server completed split of a region
49      FAILED_OPEN,    // failed to open, and won't retry any more
50      FAILED_CLOSE,   // failed to close, and won't retry any more
51      MERGING,        // server started merge a region
52      MERGED,         // server completed merge a region
53      SPLITTING_NEW,  // new region to be created when RS splits a parent
54                      // region but hasn't be created yet, or master doesn't
55                      // know it's already created
56      MERGING_NEW;    // new region to be created when RS merges two
57                      // daughter regions but hasn't be created yet, or
58                      // master doesn't know it's already created
59  
60      /**
61       * Convert to protobuf ClusterStatusProtos.RegionState.State
62       */
63      public ClusterStatusProtos.RegionState.State convert() {
64        ClusterStatusProtos.RegionState.State rs;
65        switch (this) {
66        case OFFLINE:
67          rs = ClusterStatusProtos.RegionState.State.OFFLINE;
68          break;
69        case PENDING_OPEN:
70          rs = ClusterStatusProtos.RegionState.State.PENDING_OPEN;
71          break;
72        case OPENING:
73          rs = ClusterStatusProtos.RegionState.State.OPENING;
74          break;
75        case OPEN:
76          rs = ClusterStatusProtos.RegionState.State.OPEN;
77          break;
78        case PENDING_CLOSE:
79          rs = ClusterStatusProtos.RegionState.State.PENDING_CLOSE;
80          break;
81        case CLOSING:
82          rs = ClusterStatusProtos.RegionState.State.CLOSING;
83          break;
84        case CLOSED:
85          rs = ClusterStatusProtos.RegionState.State.CLOSED;
86          break;
87        case SPLITTING:
88          rs = ClusterStatusProtos.RegionState.State.SPLITTING;
89          break;
90        case SPLIT:
91          rs = ClusterStatusProtos.RegionState.State.SPLIT;
92          break;
93        case FAILED_OPEN:
94          rs = ClusterStatusProtos.RegionState.State.FAILED_OPEN;
95          break;
96        case FAILED_CLOSE:
97          rs = ClusterStatusProtos.RegionState.State.FAILED_CLOSE;
98          break;
99        case MERGING:
100         rs = ClusterStatusProtos.RegionState.State.MERGING;
101         break;
102       case MERGED:
103         rs = ClusterStatusProtos.RegionState.State.MERGED;
104         break;
105       case SPLITTING_NEW:
106         rs = ClusterStatusProtos.RegionState.State.SPLITTING_NEW;
107         break;
108       case MERGING_NEW:
109         rs = ClusterStatusProtos.RegionState.State.MERGING_NEW;
110         break;
111       default:
112         throw new IllegalStateException("");
113       }
114       return rs;
115     }
116 
117     /**
118      * Convert a protobuf HBaseProtos.RegionState.State to a RegionState.State
119      *
120      * @return the RegionState.State
121      */
122     public static State convert(ClusterStatusProtos.RegionState.State protoState) {
123       State state;
124       switch (protoState) {
125       case OFFLINE:
126         state = OFFLINE;
127         break;
128       case PENDING_OPEN:
129         state = PENDING_OPEN;
130         break;
131       case OPENING:
132         state = OPENING;
133         break;
134       case OPEN:
135         state = OPEN;
136         break;
137       case PENDING_CLOSE:
138         state = PENDING_CLOSE;
139         break;
140       case CLOSING:
141         state = CLOSING;
142         break;
143       case CLOSED:
144         state = CLOSED;
145         break;
146       case SPLITTING:
147         state = SPLITTING;
148         break;
149       case SPLIT:
150         state = SPLIT;
151         break;
152       case FAILED_OPEN:
153         state = FAILED_OPEN;
154         break;
155       case FAILED_CLOSE:
156         state = FAILED_CLOSE;
157         break;
158       case MERGING:
159         state = MERGING;
160         break;
161       case MERGED:
162         state = MERGED;
163         break;
164       case SPLITTING_NEW:
165         state = SPLITTING_NEW;
166         break;
167       case MERGING_NEW:
168         state = MERGING_NEW;
169         break;
170       default:
171         throw new IllegalStateException("");
172       }
173       return state;
174     }
175   }
176 
177   // Many threads can update the state at the stamp at the same time
178   private final AtomicLong stamp;
179   private HRegionInfo hri;
180 
181   private volatile ServerName serverName;
182   private volatile State state;
183 
184   public RegionState() {
185     this.stamp = new AtomicLong(System.currentTimeMillis());
186   }
187 
188   public RegionState(HRegionInfo region, State state) {
189     this(region, state, System.currentTimeMillis(), null);
190   }
191 
192   public RegionState(HRegionInfo region,
193       State state, ServerName serverName) {
194     this(region, state, System.currentTimeMillis(), serverName);
195   }
196 
197   public RegionState(HRegionInfo region,
198       State state, long stamp, ServerName serverName) {
199     this.hri = region;
200     this.state = state;
201     this.stamp = new AtomicLong(stamp);
202     this.serverName = serverName;
203   }
204 
205   public void updateTimestampToNow() {
206     setTimestamp(System.currentTimeMillis());
207   }
208 
209   public State getState() {
210     return state;
211   }
212 
213   public long getStamp() {
214     return stamp.get();
215   }
216 
217   public HRegionInfo getRegion() {
218     return hri;
219   }
220 
221   public ServerName getServerName() {
222     return serverName;
223   }
224 
225   public boolean isClosing() {
226     return state == State.CLOSING;
227   }
228 
229   public boolean isClosed() {
230     return state == State.CLOSED;
231   }
232 
233   public boolean isPendingClose() {
234     return state == State.PENDING_CLOSE;
235   }
236 
237   public boolean isOpening() {
238     return state == State.OPENING;
239   }
240 
241   public boolean isOpened() {
242     return state == State.OPEN;
243   }
244 
245   public boolean isPendingOpen() {
246     return state == State.PENDING_OPEN;
247   }
248 
249   public boolean isOffline() {
250     return state == State.OFFLINE;
251   }
252 
253   public boolean isSplitting() {
254     return state == State.SPLITTING;
255   }
256 
257   public boolean isSplit() {
258     return state == State.SPLIT;
259   }
260 
261   public boolean isSplittingNew() {
262     return state == State.SPLITTING_NEW;
263   }
264 
265   public boolean isFailedOpen() {
266     return state == State.FAILED_OPEN;
267   }
268 
269   public boolean isFailedClose() {
270     return state == State.FAILED_CLOSE;
271   }
272 
273   public boolean isMerging() {
274     return state == State.MERGING;
275   }
276 
277   public boolean isMerged() {
278     return state == State.MERGED;
279   }
280 
281   public boolean isMergingNew() {
282     return state == State.MERGING_NEW;
283   }
284 
285   public boolean isOpenOrMergingOnServer(final ServerName sn) {
286     return isOnServer(sn) && (isOpened() || isMerging());
287   }
288 
289   public boolean isOpenOrMergingNewOnServer(final ServerName sn) {
290     return isOnServer(sn) && (isOpened() || isMergingNew());
291   }
292 
293   public boolean isOpenOrSplittingOnServer(final ServerName sn) {
294     return isOnServer(sn) && (isOpened() || isSplitting());
295   }
296 
297   public boolean isOpenOrSplittingNewOnServer(final ServerName sn) {
298     return isOnServer(sn) && (isOpened() || isSplittingNew());
299   }
300 
301   public boolean isPendingOpenOrOpeningOnServer(final ServerName sn) {
302     return isOnServer(sn) && isPendingOpenOrOpening();
303   }
304 
305   // Failed open is also kind of pending open
306   public boolean isPendingOpenOrOpening() {
307     return isPendingOpen() || isOpening() || isFailedOpen();
308   }
309 
310   public boolean isPendingCloseOrClosingOnServer(final ServerName sn) {
311     return isOnServer(sn) && isPendingCloseOrClosing();
312   }
313 
314   // Failed close is also kind of pending close
315   public boolean isPendingCloseOrClosing() {
316     return isPendingClose() || isClosing() || isFailedClose();
317   }
318 
319   public boolean isOnServer(final ServerName sn) {
320     return serverName != null && serverName.equals(sn);
321   }
322 
323   /**
324    * Check if a region state can transition to offline
325    */
326   public boolean isReadyToOffline() {
327     return isMerged() || isSplit() || isOffline()
328       || isSplittingNew() || isMergingNew();
329   }
330 
331   /**
332    * Check if a region state can transition to online
333    */
334   public boolean isReadyToOnline() {
335     return isOpened() || isSplittingNew() || isMergingNew();
336   }
337 
338   /**
339    * Check if a region state is one of offline states that
340    * can't transition to pending_close/closing (unassign/offline)
341    */
342   public boolean isUnassignable() {
343     return isUnassignable(state);
344   }
345 
346   /**
347    * Check if a region state is one of offline states that
348    * can't transition to pending_close/closing (unassign/offline)
349    */
350   public static boolean isUnassignable(State state) {
351     return state == State.MERGED || state == State.SPLIT || state == State.OFFLINE
352       || state == State.SPLITTING_NEW || state == State.MERGING_NEW;
353   }
354 
355   @Override
356   public String toString() {
357     return "{" + hri.getShortNameToLog()
358       + " state=" + state
359       + ", ts=" + stamp
360       + ", server=" + serverName + "}";
361   }
362 
363   /**
364    * A slower (but more easy-to-read) stringification
365    */
366   public String toDescriptiveString() {
367     long lstamp = stamp.get();
368     long relTime = System.currentTimeMillis() - lstamp;
369     
370     return hri.getRegionNameAsString()
371       + " state=" + state
372       + ", ts=" + new Date(lstamp) + " (" + (relTime/1000) + "s ago)"
373       + ", server=" + serverName;
374   }
375 
376   /**
377    * Convert a RegionState to an HBaseProtos.RegionState
378    *
379    * @return the converted HBaseProtos.RegionState
380    */
381   public ClusterStatusProtos.RegionState convert() {
382     ClusterStatusProtos.RegionState.Builder regionState = ClusterStatusProtos.RegionState.newBuilder();
383     regionState.setRegionInfo(HRegionInfo.convert(hri));
384     regionState.setState(state.convert());
385     regionState.setStamp(getStamp());
386     return regionState.build();
387   }
388 
389   /**
390    * Convert a protobuf HBaseProtos.RegionState to a RegionState
391    *
392    * @return the RegionState
393    */
394   public static RegionState convert(ClusterStatusProtos.RegionState proto) {
395     return new RegionState(HRegionInfo.convert(proto.getRegionInfo()),
396       State.convert(proto.getState()), proto.getStamp(), null);
397   }
398 
399   protected void setTimestamp(final long timestamp) {
400     stamp.set(timestamp);
401   }
402 
403   /**
404    * Check if two states are the same, except timestamp
405    */
406   @Override
407   public boolean equals(Object obj) {
408     if (this == obj) return true;
409     if (obj == null || getClass() != obj.getClass()) {
410       return false;
411     }
412     RegionState tmp = (RegionState)obj;
413     return tmp.hri.equals(hri) && tmp.state == state
414       && ((serverName != null && serverName.equals(tmp.serverName))
415         || (tmp.serverName == null && serverName == null));
416   }
417 
418   /**
419    * Don't count timestamp in hash code calculation
420    */
421   @Override
422   public int hashCode() {
423     return (serverName != null ? serverName.hashCode() * 11 : 0)
424       + hri.hashCode() + 5 * state.ordinal();
425   }
426 }