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;
019
020import edu.umd.cs.findbugs.annotations.NonNull;
021import edu.umd.cs.findbugs.annotations.Nullable;
022import java.io.IOException;
023import java.util.HashMap;
024import java.util.Map;
025import java.util.Set;
026import java.util.concurrent.ConcurrentHashMap;
027import java.util.concurrent.ConcurrentMap;
028import java.util.concurrent.locks.ReadWriteLock;
029import org.apache.hadoop.hbase.MetaTableAccessor;
030import org.apache.hadoop.hbase.TableDescriptors;
031import org.apache.hadoop.hbase.TableName;
032import org.apache.hadoop.hbase.TableNotFoundException;
033import org.apache.hadoop.hbase.client.Connection;
034import org.apache.hadoop.hbase.client.Result;
035import org.apache.hadoop.hbase.client.TableDescriptor;
036import org.apache.hadoop.hbase.client.TableState;
037import org.apache.hadoop.hbase.exceptions.IllegalArgumentIOException;
038import org.apache.hadoop.hbase.util.IdReadWriteLock;
039import org.apache.hadoop.hbase.util.ZKDataMigrator;
040import org.apache.hadoop.hbase.zookeeper.ZKUtil;
041import org.apache.hadoop.hbase.zookeeper.ZNodePaths;
042import org.apache.yetus.audience.InterfaceAudience;
043import org.apache.zookeeper.KeeperException;
044import org.slf4j.Logger;
045import org.slf4j.LoggerFactory;
046
047import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
048
049/**
050 * This is a helper class used to manage table states. This class uses hbase:meta as its store for
051 * table state so hbase:meta must be online before {@link #start()} is called.
052 */
053// TODO: Make this a guava Service
054@InterfaceAudience.Private
055public class TableStateManager {
056
057  private static final Logger LOG = LoggerFactory.getLogger(TableStateManager.class);
058  /**
059   * Set this key to false in Configuration to disable migrating table state from zookeeper so
060   * hbase:meta table.
061   */
062  private static final String MIGRATE_TABLE_STATE_FROM_ZK_KEY =
063    "hbase.migrate.table.state.from.zookeeper";
064
065  private final IdReadWriteLock<TableName> tnLock = new IdReadWriteLock<>();
066  protected final MasterServices master;
067
068  private final ConcurrentMap<TableName, TableState.State> tableName2State =
069    new ConcurrentHashMap<>();
070
071  TableStateManager(MasterServices master) {
072    this.master = master;
073  }
074
075  /**
076   * Set table state to provided. Caller should lock table on write.
077   * @param tableName table to change state for
078   * @param newState new state
079   */
080  public void setTableState(TableName tableName, TableState.State newState) throws IOException {
081    ReadWriteLock lock = tnLock.getLock(tableName);
082    lock.writeLock().lock();
083    try {
084      updateMetaState(tableName, newState);
085    } finally {
086      lock.writeLock().unlock();
087    }
088  }
089
090  public boolean isTableState(TableName tableName, TableState.State... states) {
091    try {
092      TableState tableState = getTableState(tableName);
093      return tableState.isInStates(states);
094    } catch (IOException e) {
095      LOG.error("Unable to get table " + tableName + " state", e);
096      // XXX: is it safe to just return false here?
097      return false;
098    }
099  }
100
101  public void setDeletedTable(TableName tableName) throws IOException {
102    if (tableName.equals(TableName.META_TABLE_NAME)) {
103      // Can't delete the hbase:meta table.
104      return;
105    }
106    ReadWriteLock lock = tnLock.getLock(tableName);
107    lock.writeLock().lock();
108    try {
109      MetaTableAccessor.deleteTableState(master.getConnection(), tableName);
110      metaStateDeleted(tableName);
111    } finally {
112      tableName2State.remove(tableName);
113      lock.writeLock().unlock();
114    }
115  }
116
117  public boolean isTablePresent(TableName tableName) throws IOException {
118    ReadWriteLock lock = tnLock.getLock(tableName);
119    lock.readLock().lock();
120    try {
121      return readMetaState(tableName) != null;
122    } finally {
123      lock.readLock().unlock();
124    }
125  }
126
127  /**
128   * Return all tables in given states.
129   * @param states filter by states
130   * @return tables in given states
131   */
132  Set<TableName> getTablesInStates(TableState.State... states) throws IOException {
133    // Only be called in region normalizer, will not use cache.
134    final Set<TableName> rv = Sets.newHashSet();
135    MetaTableAccessor.fullScanTables(master.getConnection(), new MetaTableAccessor.Visitor() {
136      @Override
137      public boolean visit(Result r) throws IOException {
138        TableState tableState = MetaTableAccessor.getTableState(r);
139        if (tableState != null && tableState.inStates(states)) {
140          rv.add(tableState.getTableName());
141        }
142        return true;
143      }
144    });
145    return rv;
146  }
147
148  @NonNull
149  public TableState getTableState(TableName tableName) throws IOException {
150    ReadWriteLock lock = tnLock.getLock(tableName);
151    lock.readLock().lock();
152    try {
153      TableState currentState = readMetaState(tableName);
154      if (currentState == null) {
155        throw new TableNotFoundException("No state found for " + tableName);
156      }
157      return currentState;
158    } finally {
159      lock.readLock().unlock();
160    }
161  }
162
163  private void updateMetaState(TableName tableName, TableState.State newState) throws IOException {
164    if (tableName.equals(TableName.META_TABLE_NAME)) {
165      if (TableState.State.DISABLING.equals(newState) ||
166          TableState.State.DISABLED.equals(newState)) {
167        throw new IllegalArgumentIOException("Cannot disable meta table; " + newState);
168      }
169      // Otherwise, just return; no need to set ENABLED on meta -- it is always ENABLED.
170      return;
171    }
172    boolean succ = false;
173    try {
174      MetaTableAccessor.updateTableState(master.getConnection(), tableName, newState);
175      tableName2State.put(tableName, newState);
176      succ = true;
177    } finally {
178      if (!succ) {
179        this.tableName2State.remove(tableName);
180      }
181    }
182    metaStateUpdated(tableName, newState);
183  }
184
185  protected void metaStateUpdated(TableName tableName, TableState.State newState)
186      throws IOException {
187  }
188
189  protected void metaStateDeleted(TableName tableName) throws IOException {
190  }
191
192  @Nullable
193  private TableState readMetaState(TableName tableName) throws IOException {
194    TableState.State state = tableName2State.get(tableName);
195    if (state != null) {
196      return new TableState(tableName, state);
197    }
198    TableState tableState = MetaTableAccessor.getTableState(master.getConnection(), tableName);
199    if (tableState != null) {
200      tableName2State.putIfAbsent(tableName, tableState.getState());
201    }
202    return tableState;
203  }
204
205  public void start() throws IOException {
206    migrateZooKeeper();
207    fixTableStates(master.getTableDescriptors(), master.getConnection());
208  }
209
210  private void fixTableStates(TableDescriptors tableDescriptors, Connection connection)
211      throws IOException {
212    Map<String, TableState> states = new HashMap<>();
213    // NOTE: Full hbase:meta table scan!
214    MetaTableAccessor.fullScanTables(connection, new MetaTableAccessor.Visitor() {
215      @Override
216      public boolean visit(Result r) throws IOException {
217        TableState state = MetaTableAccessor.getTableState(r);
218        states.put(state.getTableName().getNameAsString(), state);
219        return true;
220      }
221    });
222    for (TableDescriptor tableDesc : tableDescriptors.getAll().values()) {
223      TableName tableName = tableDesc.getTableName();
224      if (TableName.isMetaTableName(tableName)) {
225        // This table is always enabled. No fixup needed. No entry in hbase:meta needed.
226        // Call through to fixTableState though in case a super class wants to do something.
227        fixTableState(new TableState(tableName, TableState.State.ENABLED));
228        continue;
229      }
230      TableState tableState = states.get(tableName.getNameAsString());
231      if (tableState == null) {
232        LOG.warn(tableName + " has no table state in hbase:meta, assuming ENABLED");
233        MetaTableAccessor.updateTableState(connection, tableName, TableState.State.ENABLED);
234        fixTableState(new TableState(tableName, TableState.State.ENABLED));
235        tableName2State.put(tableName, TableState.State.ENABLED);
236      } else {
237        fixTableState(tableState);
238        tableName2State.put(tableName, tableState.getState());
239      }
240    }
241  }
242
243  /**
244   * For subclasses in case they want to do fixup post hbase:meta.
245   */
246  protected void fixTableState(TableState tableState) throws IOException {
247  }
248
249  /**
250   * This code is for case where a hbase2 Master is starting for the first time. ZooKeeper is where
251   * we used to keep table state. On first startup, read zookeeper and update hbase:meta with the
252   * table states found in zookeeper. This is tricky as we'll do this check every time we startup
253   * until mirroring is disabled. See the {@link #MIGRATE_TABLE_STATE_FROM_ZK_KEY} flag. Original
254   * form of this migration came in with HBASE-13032. It deleted all znodes when done. We can't do
255   * that if we want to support hbase-1.x clients who need to be able to read table state out of zk.
256   * See {@link MirroringTableStateManager}.
257   * @deprecated Since 2.0.0. Remove in hbase-3.0.0.
258   */
259  @Deprecated
260  private void migrateZooKeeper() throws IOException {
261    if (!this.master.getConfiguration().getBoolean(MIGRATE_TABLE_STATE_FROM_ZK_KEY, true)) {
262      return;
263    }
264    try {
265      for (Map.Entry<TableName, TableState.State> entry : ZKDataMigrator
266        .queryForTableStates(this.master.getZooKeeper()).entrySet()) {
267        if (this.master.getTableDescriptors().get(entry.getKey()) == null) {
268          deleteZooKeeper(entry.getKey());
269          LOG.info("Purged table state entry from zookeepr for table not in hbase:meta: " +
270            entry.getKey());
271          continue;
272        }
273        TableState ts = null;
274        try {
275          ts = getTableState(entry.getKey());
276        } catch (TableNotFoundException e) {
277          // This can happen; table exists but no TableState.
278        }
279        if (ts == null) {
280          TableState.State zkstate = entry.getValue();
281          // Only migrate if it is an enable or disabled table. If in-between -- ENABLING or
282          // DISABLING then we have a problem; we are starting up an hbase-2 on a cluster with
283          // RIT. It is going to be rough!
284          if (zkstate.equals(TableState.State.ENABLED) ||
285            zkstate.equals(TableState.State.DISABLED)) {
286            LOG.info("Migrating table state from zookeeper to hbase:meta; tableName=" +
287              entry.getKey() + ", state=" + entry.getValue());
288            updateMetaState(entry.getKey(), entry.getValue());
289          } else {
290            LOG.warn("Table={} has no state and zookeeper state is in-between={} (neither " +
291              "ENABLED or DISABLED); NOT MIGRATING table state", entry.getKey(), zkstate);
292          }
293        }
294        // What if the table states disagree? Defer to the hbase:meta setting rather than have the
295        // hbase-1.x support prevail.
296      }
297    } catch (KeeperException | InterruptedException e) {
298      LOG.warn("Failed reading table state from zookeeper", e);
299    }
300  }
301
302  /**
303   * Utility method that knows how to delete the old hbase-1.x table state znode. Used also by the
304   * Mirroring subclass.
305   * @deprecated Since 2.0.0. To be removed in hbase-3.0.0.
306   */
307  @Deprecated
308  protected void deleteZooKeeper(TableName tableName) {
309    try {
310      // Delete from ZooKeeper
311      String znode = ZNodePaths.joinZNode(this.master.getZooKeeper().getZNodePaths().tableZNode,
312        tableName.getNameAsString());
313      ZKUtil.deleteNodeFailSilent(this.master.getZooKeeper(), znode);
314    } catch (KeeperException e) {
315      LOG.warn("Failed deleting table state from zookeeper", e);
316    }
317  }
318}