View Javadoc

1   /*
2    * Copyright The Apache Software Foundation
3    *
4    * Licensed to the Apache Software Foundation (ASF) under one
5    * or more contributor license agreements.  See the NOTICE file
6    * distributed with this work for additional information
7    * regarding copyright ownership.  The ASF licenses this file
8    * to you under the Apache License, Version 2.0 (the
9    * "License"); you may not use this file except in compliance
10   * with the License.  You may obtain a copy of the License at
11   *
12   *     http://www.apache.org/licenses/LICENSE-2.0
13   *
14   * Unless required by applicable law or agreed to in writing, software
15   * distributed under the License is distributed on an "AS IS" BASIS,
16   * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
17   * See the License for the specific language governing permissions and
18   * limitations under the License.
19   */
20  package org.apache.hadoop.hbase.master;
21  
22  import java.io.IOException;
23  import java.io.InterruptedIOException;
24  import java.util.List;
25  
26  import org.apache.commons.logging.Log;
27  import org.apache.commons.logging.LogFactory;
28  import org.apache.hadoop.classification.InterfaceAudience;
29  import org.apache.hadoop.conf.Configuration;
30  import org.apache.hadoop.hbase.TableName;
31  import org.apache.hadoop.hbase.InterProcessLock;
32  import org.apache.hadoop.hbase.InterProcessLock.MetadataHandler;
33  import org.apache.hadoop.hbase.InterProcessReadWriteLock;
34  import org.apache.hadoop.hbase.ServerName;
35  import org.apache.hadoop.hbase.exceptions.LockTimeoutException;
36  import org.apache.hadoop.hbase.protobuf.ProtobufUtil;
37  import org.apache.hadoop.hbase.protobuf.generated.ZooKeeperProtos;
38  import org.apache.hadoop.hbase.util.Bytes;
39  import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
40  import org.apache.hadoop.hbase.zookeeper.ZKUtil;
41  import org.apache.hadoop.hbase.zookeeper.ZooKeeperWatcher;
42  import org.apache.hadoop.hbase.zookeeper.lock.ZKInterProcessReadWriteLock;
43  import org.apache.zookeeper.KeeperException;
44  
45  import com.google.protobuf.InvalidProtocolBufferException;
46  
47  /**
48   * A manager for distributed table level locks.
49   */
50  @InterfaceAudience.Private
51  public abstract class TableLockManager {
52  
53    private static final Log LOG = LogFactory.getLog(TableLockManager.class);
54  
55    /** Configuration key for enabling table-level locks for schema changes */
56    public static final String TABLE_LOCK_ENABLE =
57      "hbase.table.lock.enable";
58  
59    /** by default we should enable table-level locks for schema changes */
60    private static final boolean DEFAULT_TABLE_LOCK_ENABLE = true;
61  
62    /** Configuration key for time out for trying to acquire table locks */
63    protected static final String TABLE_WRITE_LOCK_TIMEOUT_MS =
64      "hbase.table.write.lock.timeout.ms";
65  
66    /** Configuration key for time out for trying to acquire table locks */
67    protected static final String TABLE_READ_LOCK_TIMEOUT_MS =
68      "hbase.table.read.lock.timeout.ms";
69  
70    protected static final long DEFAULT_TABLE_WRITE_LOCK_TIMEOUT_MS =
71      600 * 1000; //10 min default
72  
73    protected static final long DEFAULT_TABLE_READ_LOCK_TIMEOUT_MS =
74      600 * 1000; //10 min default
75  
76    public static final String TABLE_LOCK_EXPIRE_TIMEOUT = "hbase.table.lock.expire.ms";
77  
78    public static final long DEFAULT_TABLE_LOCK_EXPIRE_TIMEOUT_MS =
79        600 * 1000; //10 min default
80  
81    /**
82     * A distributed lock for a table.
83     */
84    @InterfaceAudience.Private
85    public interface TableLock {
86      /**
87       * Acquire the lock, with the configured lock timeout.
88       * @throws LockTimeoutException If unable to acquire a lock within a specified
89       * time period (if any)
90       * @throws IOException If unrecoverable error occurs
91       */
92      void acquire() throws IOException;
93  
94      /**
95       * Release the lock already held.
96       * @throws IOException If there is an unrecoverable error releasing the lock
97       */
98      void release() throws IOException;
99    }
100 
101   /**
102    * Returns a TableLock for locking the table for exclusive access
103    * @param tableName Table to lock
104    * @param purpose Human readable reason for locking the table
105    * @return A new TableLock object for acquiring a write lock
106    */
107   public abstract TableLock writeLock(TableName tableName, String purpose);
108 
109   /**
110    * Returns a TableLock for locking the table for shared access among read-lock holders
111    * @param tableName Table to lock
112    * @param purpose Human readable reason for locking the table
113    * @return A new TableLock object for acquiring a read lock
114    */
115   public abstract TableLock readLock(TableName tableName, String purpose);
116 
117   /**
118    * Visits all table locks(read and write), and lock attempts with the given callback
119    * MetadataHandler.
120    * @param handler the metadata handler to call
121    * @throws IOException If there is an unrecoverable error
122    */
123   public abstract void visitAllLocks(MetadataHandler handler) throws IOException;
124 
125   /**
126    * Force releases all table locks(read and write) that have been held longer than
127    * "hbase.table.lock.expire.ms". Assumption is that the clock skew between zookeeper
128    * and this servers is negligible.
129    * The behavior of the lock holders still thinking that they have the lock is undefined.
130    * @throws IOException If there is an unrecoverable error
131    */
132   public abstract void reapAllExpiredLocks() throws IOException;
133 
134   /**
135    * Force releases table write locks and lock attempts even if this thread does
136    * not own the lock. The behavior of the lock holders still thinking that they
137    * have the lock is undefined. This should be used carefully and only when
138    * we can ensure that all write-lock holders have died. For example if only
139    * the master can hold write locks, then we can reap it's locks when the backup
140    * master starts.
141    * @throws IOException If there is an unrecoverable error
142    */
143   public abstract void reapWriteLocks() throws IOException;
144 
145   /**
146    * Called after a table has been deleted, and after the table lock is  released.
147    * TableLockManager should do cleanup for the table state.
148    * @param tableName name of the table
149    * @throws IOException If there is an unrecoverable error releasing the lock
150    */
151   public abstract void tableDeleted(TableName tableName)
152       throws IOException;
153 
154   /**
155    * Creates and returns a TableLockManager according to the configuration
156    */
157   public static TableLockManager createTableLockManager(Configuration conf,
158       ZooKeeperWatcher zkWatcher, ServerName serverName) {
159     // Initialize table level lock manager for schema changes, if enabled.
160     if (conf.getBoolean(TABLE_LOCK_ENABLE,
161         DEFAULT_TABLE_LOCK_ENABLE)) {
162       long writeLockTimeoutMs = conf.getLong(TABLE_WRITE_LOCK_TIMEOUT_MS,
163           DEFAULT_TABLE_WRITE_LOCK_TIMEOUT_MS);
164       long readLockTimeoutMs = conf.getLong(TABLE_READ_LOCK_TIMEOUT_MS,
165           DEFAULT_TABLE_READ_LOCK_TIMEOUT_MS);
166       long lockExpireTimeoutMs = conf.getLong(TABLE_LOCK_EXPIRE_TIMEOUT,
167           DEFAULT_TABLE_LOCK_EXPIRE_TIMEOUT_MS);
168 
169       return new ZKTableLockManager(zkWatcher, serverName, writeLockTimeoutMs, readLockTimeoutMs, lockExpireTimeoutMs);
170     }
171 
172     return new NullTableLockManager();
173   }
174 
175   /**
176    * A null implementation
177    */
178   @InterfaceAudience.Private
179   public static class NullTableLockManager extends TableLockManager {
180     static class NullTableLock implements TableLock {
181       @Override
182       public void acquire() throws IOException {
183       }
184       @Override
185       public void release() throws IOException {
186       }
187     }
188     @Override
189     public TableLock writeLock(TableName tableName, String purpose) {
190       return new NullTableLock();
191     }
192     @Override
193     public TableLock readLock(TableName tableName, String purpose) {
194       return new NullTableLock();
195     }
196     @Override
197     public void reapAllExpiredLocks() throws IOException {
198     }
199     @Override
200     public void reapWriteLocks() throws IOException {
201     }
202     @Override
203     public void tableDeleted(TableName tableName) throws IOException {
204     }
205     @Override
206     public void visitAllLocks(MetadataHandler handler) throws IOException {
207     }
208   }
209 
210   /** Public for hbck */
211   public static ZooKeeperProtos.TableLock fromBytes(byte[] bytes) {
212     int pblen = ProtobufUtil.lengthOfPBMagic();
213     if (bytes == null || bytes.length < pblen) {
214       return null;
215     }
216     try {
217       ZooKeeperProtos.TableLock data = ZooKeeperProtos.TableLock.newBuilder().mergeFrom(
218           bytes, pblen, bytes.length - pblen).build();
219       return data;
220     } catch (InvalidProtocolBufferException ex) {
221       LOG.warn("Exception in deserialization", ex);
222     }
223     return null;
224   }
225 
226   /**
227    * ZooKeeper based TableLockManager
228    */
229   @InterfaceAudience.Private
230   private static class ZKTableLockManager extends TableLockManager {
231 
232     private static final MetadataHandler METADATA_HANDLER = new MetadataHandler() {
233       @Override
234       public void handleMetadata(byte[] ownerMetadata) {
235         if (!LOG.isDebugEnabled()) {
236           return;
237         }
238         ZooKeeperProtos.TableLock data = fromBytes(ownerMetadata);
239         if (data == null) {
240           return;
241         }
242         LOG.debug("Table is locked by " +
243             String.format("[tableName=%s, lockOwner=%s, threadId=%s, " +
244                 "purpose=%s, isShared=%s, createTime=%s]", Bytes.toString(data.getTableName().toByteArray()),
245                 ProtobufUtil.toServerName(data.getLockOwner()), data.getThreadId(),
246                 data.getPurpose(), data.getIsShared(), data.getCreateTime()));
247       }
248     };
249 
250     private static class TableLockImpl implements TableLock {
251       long lockTimeoutMs;
252       TableName tableName;
253       InterProcessLock lock;
254       boolean isShared;
255       ZooKeeperWatcher zkWatcher;
256       ServerName serverName;
257       String purpose;
258 
259       public TableLockImpl(TableName tableName, ZooKeeperWatcher zkWatcher,
260           ServerName serverName, long lockTimeoutMs, boolean isShared, String purpose) {
261         this.tableName = tableName;
262         this.zkWatcher = zkWatcher;
263         this.serverName = serverName;
264         this.lockTimeoutMs = lockTimeoutMs;
265         this.isShared = isShared;
266         this.purpose = purpose;
267       }
268 
269       @Override
270       public void acquire() throws IOException {
271         if (LOG.isTraceEnabled()) {
272           LOG.trace("Attempt to acquire table " + (isShared ? "read" : "write") +
273             " lock on: " + tableName + " for:" + purpose);
274         }
275 
276         lock = createTableLock();
277         try {
278           if (lockTimeoutMs == -1) {
279             // Wait indefinitely
280             lock.acquire();
281           } else {
282             if (!lock.tryAcquire(lockTimeoutMs)) {
283               throw new LockTimeoutException("Timed out acquiring " +
284                 (isShared ? "read" : "write") + "lock for table:" + tableName +
285                 "for:" + purpose + " after " + lockTimeoutMs + " ms.");
286             }
287           }
288         } catch (InterruptedException e) {
289           LOG.warn("Interrupted acquiring a lock for " + tableName, e);
290           Thread.currentThread().interrupt();
291           throw new InterruptedIOException("Interrupted acquiring a lock");
292         }
293         if (LOG.isTraceEnabled()) LOG.trace("Acquired table " + (isShared ? "read" : "write")
294             + " lock on " + tableName + " for " + purpose);
295       }
296 
297       @Override
298       public void release() throws IOException {
299         if (LOG.isTraceEnabled()) {
300           LOG.trace("Attempt to release table " + (isShared ? "read" : "write")
301               + " lock on " + tableName);
302         }
303         if (lock == null) {
304           throw new IllegalStateException("Table " + tableName +
305             " is not locked!");
306         }
307 
308         try {
309           lock.release();
310         } catch (InterruptedException e) {
311           LOG.warn("Interrupted while releasing a lock for " + tableName);
312           Thread.currentThread().interrupt();
313           throw new InterruptedIOException();
314         }
315         if (LOG.isTraceEnabled()) {
316           LOG.trace("Released table lock on " + tableName);
317         }
318       }
319 
320       private InterProcessLock createTableLock() {
321         String tableLockZNode = ZKUtil.joinZNode(zkWatcher.tableLockZNode,
322             tableName.getNameAsString());
323 
324         ZooKeeperProtos.TableLock data = ZooKeeperProtos.TableLock.newBuilder()
325           .setTableName(ProtobufUtil.toProtoTableName(tableName))
326           .setLockOwner(ProtobufUtil.toServerName(serverName))
327           .setThreadId(Thread.currentThread().getId())
328           .setPurpose(purpose)
329           .setIsShared(isShared)
330           .setCreateTime(EnvironmentEdgeManager.currentTimeMillis()).build();
331         byte[] lockMetadata = toBytes(data);
332 
333         InterProcessReadWriteLock lock = new ZKInterProcessReadWriteLock(zkWatcher, tableLockZNode,
334           METADATA_HANDLER);
335         return isShared ? lock.readLock(lockMetadata) : lock.writeLock(lockMetadata);
336       }
337     }
338 
339     private static byte[] toBytes(ZooKeeperProtos.TableLock data) {
340       return ProtobufUtil.prependPBMagic(data.toByteArray());
341     }
342 
343     private final ServerName serverName;
344     private final ZooKeeperWatcher zkWatcher;
345     private final long writeLockTimeoutMs;
346     private final long readLockTimeoutMs;
347     private final long lockExpireTimeoutMs;
348 
349     /**
350      * Initialize a new manager for table-level locks.
351      * @param zkWatcher
352      * @param serverName Address of the server responsible for acquiring and
353      * releasing the table-level locks
354      * @param writeLockTimeoutMs Timeout (in milliseconds) for acquiring a write lock for a
355      * given table, or -1 for no timeout
356      * @param readLockTimeoutMs Timeout (in milliseconds) for acquiring a read lock for a
357      * given table, or -1 for no timeout
358      */
359     public ZKTableLockManager(ZooKeeperWatcher zkWatcher,
360       ServerName serverName, long writeLockTimeoutMs, long readLockTimeoutMs, long lockExpireTimeoutMs) {
361       this.zkWatcher = zkWatcher;
362       this.serverName = serverName;
363       this.writeLockTimeoutMs = writeLockTimeoutMs;
364       this.readLockTimeoutMs = readLockTimeoutMs;
365       this.lockExpireTimeoutMs = lockExpireTimeoutMs;
366     }
367 
368     @Override
369     public TableLock writeLock(TableName tableName, String purpose) {
370       return new TableLockImpl(tableName, zkWatcher,
371           serverName, writeLockTimeoutMs, false, purpose);
372     }
373 
374     public TableLock readLock(TableName tableName, String purpose) {
375       return new TableLockImpl(tableName, zkWatcher,
376           serverName, readLockTimeoutMs, true, purpose);
377     }
378 
379     public void visitAllLocks(MetadataHandler handler) throws IOException {
380       for (String tableName : getTableNames()) {
381         String tableLockZNode = ZKUtil.joinZNode(zkWatcher.tableLockZNode, tableName);
382         ZKInterProcessReadWriteLock lock = new ZKInterProcessReadWriteLock(
383             zkWatcher, tableLockZNode, null);
384         lock.readLock(null).visitLocks(handler);
385         lock.writeLock(null).visitLocks(handler);
386       }
387     }
388 
389     private List<String> getTableNames() throws IOException {
390 
391       List<String> tableNames;
392       try {
393         tableNames = ZKUtil.listChildrenNoWatch(zkWatcher, zkWatcher.tableLockZNode);
394       } catch (KeeperException e) {
395         LOG.error("Unexpected ZooKeeper error when listing children", e);
396         throw new IOException("Unexpected ZooKeeper exception", e);
397       }
398       return tableNames;
399     }
400 
401     @Override
402     public void reapWriteLocks() throws IOException {
403       //get the table names
404       try {
405         for (String tableName : getTableNames()) {
406           String tableLockZNode = ZKUtil.joinZNode(zkWatcher.tableLockZNode, tableName);
407           ZKInterProcessReadWriteLock lock = new ZKInterProcessReadWriteLock(
408               zkWatcher, tableLockZNode, null);
409           lock.writeLock(null).reapAllLocks();
410         }
411       } catch (IOException ex) {
412         throw ex;
413       } catch (Exception ex) {
414         LOG.warn("Caught exception while reaping table write locks", ex);
415       }
416     }
417 
418     @Override
419     public void reapAllExpiredLocks() throws IOException {
420       //get the table names
421       try {
422         for (String tableName : getTableNames()) {
423           String tableLockZNode = ZKUtil.joinZNode(zkWatcher.tableLockZNode, tableName);
424           ZKInterProcessReadWriteLock lock = new ZKInterProcessReadWriteLock(
425               zkWatcher, tableLockZNode, null);
426           lock.readLock(null).reapExpiredLocks(lockExpireTimeoutMs);
427           lock.writeLock(null).reapExpiredLocks(lockExpireTimeoutMs);
428         }
429       } catch (IOException ex) {
430         throw ex;
431       } catch (Exception ex) {
432         throw new IOException(ex);
433       }
434     }
435 
436     @Override
437     public void tableDeleted(TableName tableName) throws IOException {
438       //table write lock from DeleteHandler is already released, just delete the parent znode
439       String tableNameStr = tableName.getNameAsString();
440       String tableLockZNode = ZKUtil.joinZNode(zkWatcher.tableLockZNode, tableNameStr);
441       try {
442         ZKUtil.deleteNode(zkWatcher, tableLockZNode);
443       } catch (KeeperException ex) {
444         if (ex.code() == KeeperException.Code.NOTEMPTY) {
445           //we might get this in rare occasions where a CREATE table or some other table operation
446           //is waiting to acquire the lock. In this case, parent znode won't be deleted.
447           LOG.warn("Could not delete the znode for table locks because NOTEMPTY: "
448               + tableLockZNode);
449           return;
450         }
451         throw new IOException(ex);
452       }
453     }
454   }
455 }