001/**
002 *
003 * Licensed to the Apache Software Foundation (ASF) under one
004 * or more contributor license agreements.  See the NOTICE file
005 * distributed with this work for additional information
006 * regarding copyright ownership.  The ASF licenses this file
007 * to you under the Apache License, Version 2.0 (the
008 * "License"); you may not use this file except in compliance
009 * with the License.  You may obtain a copy of the License at
010 *
011 *     http://www.apache.org/licenses/LICENSE-2.0
012 *
013 * Unless required by applicable law or agreed to in writing, software
014 * distributed under the License is distributed on an "AS IS" BASIS,
015 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
016 * See the License for the specific language governing permissions and
017 * limitations under the License.
018 */
019package org.apache.hadoop.hbase.master.locking;
020
021import java.io.IOException;
022import java.util.concurrent.CountDownLatch;
023import java.util.concurrent.TimeUnit;
024
025import org.apache.hadoop.hbase.TableName;
026import org.apache.hadoop.hbase.client.RegionInfo;
027import org.apache.hadoop.hbase.master.HMaster;
028import org.apache.hadoop.hbase.procedure2.LockType;
029import org.apache.hadoop.hbase.util.NonceKey;
030import org.apache.yetus.audience.InterfaceAudience;
031import org.slf4j.Logger;
032import org.slf4j.LoggerFactory;
033
034/**
035 * Functions to acquire lock on table/namespace/regions.
036 */
037@InterfaceAudience.Private
038public final class LockManager {
039  private static final Logger LOG = LoggerFactory.getLogger(LockManager.class);
040  private final HMaster master;
041  private final RemoteLocks remoteLocks;
042
043  public LockManager(HMaster master) {
044    this.master = master;
045    this.remoteLocks = new RemoteLocks();
046  }
047
048  public RemoteLocks remoteLocks() {
049    return remoteLocks;
050  }
051
052  public MasterLock createMasterLock(final String namespace,
053      final LockType type, final String description) {
054    return new MasterLock(namespace, type, description);
055  }
056
057  public MasterLock createMasterLock(final TableName tableName,
058      final LockType type, final String description) {
059    return new MasterLock(tableName, type, description);
060  }
061
062  public MasterLock createMasterLock(final RegionInfo[] regionInfos, final String description) {
063    return new MasterLock(regionInfos, description);
064  }
065
066  private void submitProcedure(final LockProcedure proc, final NonceKey nonceKey) {
067    proc.setOwner(master.getMasterProcedureExecutor().getEnvironment().getRequestUser());
068    master.getMasterProcedureExecutor().submitProcedure(proc, nonceKey);
069  }
070
071  /**
072   * Locks on namespace/table/regions.
073   * Underneath, uses procedure framework and queues a {@link LockProcedure} which waits in a
074   * queue until scheduled.
075   * Use this lock instead LockManager.remoteLocks() for MASTER ONLY operations for two advantages:
076   * - no need of polling on LockProcedure to check if lock was acquired.
077   * - Generous timeout for lock preemption (default 10 min), no need to spawn thread for heartbeats.
078   * (timeout configuration {@link LockProcedure#DEFAULT_LOCAL_MASTER_LOCKS_TIMEOUT_MS}).
079   */
080  public class MasterLock {
081    private final String namespace;
082    private final TableName tableName;
083    private final RegionInfo[] regionInfos;
084    private final LockType type;
085    private final String description;
086
087    private LockProcedure proc = null;
088
089    public MasterLock(final String namespace,
090        final LockType type, final String description) {
091      this.namespace = namespace;
092      this.tableName = null;
093      this.regionInfos = null;
094      this.type = type;
095      this.description = description;
096    }
097
098    public MasterLock(final TableName tableName,
099        final LockType type, final String description) {
100      this.namespace = null;
101      this.tableName = tableName;
102      this.regionInfos = null;
103      this.type = type;
104      this.description = description;
105    }
106
107    public MasterLock(final RegionInfo[] regionInfos, final String description) {
108      this.namespace = null;
109      this.tableName = null;
110      this.regionInfos = regionInfos;
111      this.type = LockType.EXCLUSIVE;
112      this.description = description;
113    }
114
115    /**
116     * Acquire the lock, waiting indefinitely until the lock is released or
117     * the thread is interrupted.
118     * @throws InterruptedException If current thread is interrupted while
119     *                              waiting for the lock
120     */
121    public boolean acquire() throws InterruptedException {
122      return tryAcquire(0);
123    }
124
125    /**
126     * Acquire the lock within a wait time.
127     * @param timeoutMs The maximum time (in milliseconds) to wait for the lock,
128     *                  0 to wait indefinitely
129     * @return True if the lock was acquired, false if waiting time elapsed
130     *         before the lock was acquired
131     * @throws InterruptedException If the thread is interrupted while waiting to
132     *                              acquire the lock
133     */
134    public boolean tryAcquire(final long timeoutMs) throws InterruptedException {
135      if (proc != null && proc.isLocked()) {
136        return true;
137      }
138      // Use new condition and procedure every time lock is requested.
139      final CountDownLatch lockAcquireLatch = new CountDownLatch(1);
140      if (regionInfos != null) {
141        proc = new LockProcedure(master.getConfiguration(), regionInfos, type,
142            description, lockAcquireLatch);
143      } else if (tableName != null) {
144        proc = new LockProcedure(master.getConfiguration(), tableName, type,
145            description, lockAcquireLatch);
146      } else if (namespace != null) {
147        proc = new LockProcedure(master.getConfiguration(), namespace, type,
148            description, lockAcquireLatch);
149      } else {
150        throw new UnsupportedOperationException("no namespace/table/region provided");
151      }
152
153      // The user of a MasterLock should be 'hbase', the only case where this is not true
154      // is if from inside a coprocessor we try to take a master lock (which should be avoided)
155      proc.setOwner(master.getMasterProcedureExecutor().getEnvironment().getRequestUser());
156      master.getMasterProcedureExecutor().submitProcedure(proc);
157
158      long deadline = (timeoutMs > 0) ? System.currentTimeMillis() + timeoutMs : Long.MAX_VALUE;
159      while (deadline >= System.currentTimeMillis() && !proc.isLocked()) {
160        try {
161          lockAcquireLatch.await(deadline - System.currentTimeMillis(), TimeUnit.MILLISECONDS);
162        } catch (InterruptedException e) {
163          LOG.info("InterruptedException when waiting for lock: " + proc.toString());
164          // kind of weird, releasing a lock which is not locked. This is to make the procedure
165          // finish immediately whenever it gets scheduled so that it doesn't hold the lock.
166          release();
167          throw e;
168        }
169      }
170      if (!proc.isLocked()) {
171        LOG.info("Timed out waiting to acquire procedure lock: " + proc.toString());
172        release();
173        return false;
174      }
175      return true;
176    }
177
178    /**
179     * Release the lock.
180     * No-op if the lock was never acquired.
181     */
182    public void release() {
183      if (proc != null) {
184        proc.unlock(master.getMasterProcedureExecutor().getEnvironment());
185      }
186      proc = null;
187    }
188
189    @Override
190    public String toString() {
191      return "MasterLock: proc = " + proc.toString();
192    }
193
194    LockProcedure getProc() {
195      return proc;
196    }
197  }
198
199  /**
200   * Locks on namespace/table/regions for remote operations.
201   * Since remote operations are unreliable and the client/RS may die anytime and never release
202   * locks, regular heartbeats are required to keep the lock held.
203   */
204  public class RemoteLocks {
205    public long requestNamespaceLock(final String namespace, final LockType type,
206        final String description, final NonceKey nonceKey)
207        throws IllegalArgumentException, IOException {
208      master.getMasterCoprocessorHost().preRequestLock(namespace, null, null, type, description);
209      final LockProcedure proc = new LockProcedure(master.getConfiguration(), namespace,
210          type, description, null);
211      submitProcedure(proc, nonceKey);
212      master.getMasterCoprocessorHost().postRequestLock(namespace, null, null, type, description);
213      return proc.getProcId();
214    }
215
216    public long requestTableLock(final TableName tableName, final LockType type,
217        final String description, final NonceKey nonceKey)
218        throws IllegalArgumentException, IOException {
219      master.getMasterCoprocessorHost().preRequestLock(null, tableName, null, type, description);
220      final LockProcedure proc = new LockProcedure(master.getConfiguration(), tableName,
221          type, description, null);
222      submitProcedure(proc, nonceKey);
223      master.getMasterCoprocessorHost().postRequestLock(null, tableName, null, type, description);
224      return proc.getProcId();
225    }
226
227    /**
228     * @throws IllegalArgumentException if all regions are not from same table.
229     */
230    public long requestRegionsLock(final RegionInfo[] regionInfos, final String description,
231        final NonceKey nonceKey)
232    throws IllegalArgumentException, IOException {
233      master.getMasterCoprocessorHost().preRequestLock(null, null, regionInfos,
234            LockType.EXCLUSIVE, description);
235      final LockProcedure proc = new LockProcedure(master.getConfiguration(), regionInfos,
236          LockType.EXCLUSIVE, description, null);
237      submitProcedure(proc, nonceKey);
238      master.getMasterCoprocessorHost().postRequestLock(null, null, regionInfos,
239            LockType.EXCLUSIVE, description);
240      return proc.getProcId();
241    }
242
243    /**
244     * @param keepAlive if false, release the lock.
245     * @return true, if procedure is found and it has the lock; else false.
246     */
247    public boolean lockHeartbeat(final long procId, final boolean keepAlive) throws IOException {
248      final LockProcedure proc = master.getMasterProcedureExecutor()
249        .getProcedure(LockProcedure.class, procId);
250      if (proc == null) return false;
251
252      master.getMasterCoprocessorHost().preLockHeartbeat(proc, keepAlive);
253
254      proc.updateHeartBeat();
255      if (!keepAlive) {
256        proc.unlock(master.getMasterProcedureExecutor().getEnvironment());
257      }
258
259      master.getMasterCoprocessorHost().postLockHeartbeat(proc, keepAlive);
260
261      return proc.isLocked();
262    }
263  }
264}