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