HBASE-21154 Remove hbase:namespace table; fold it into hbase:meta
This commit is contained in:
parent
a0e3cb6c0c
commit
1acbd36c90
|
@ -220,11 +220,28 @@ public class TableDescriptorBuilder {
|
||||||
RESERVED_KEYWORDS.add(IS_META_KEY);
|
RESERVED_KEYWORDS.add(IS_META_KEY);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @deprecated namespace table has been folded into the ns family in meta table, do not use this
|
||||||
|
* any more.
|
||||||
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
|
@Deprecated
|
||||||
public final static String NAMESPACE_FAMILY_INFO = "info";
|
public final static String NAMESPACE_FAMILY_INFO = "info";
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @deprecated namespace table has been folded into the ns family in meta table, do not use this
|
||||||
|
* any more.
|
||||||
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
|
@Deprecated
|
||||||
public final static byte[] NAMESPACE_FAMILY_INFO_BYTES = Bytes.toBytes(NAMESPACE_FAMILY_INFO);
|
public final static byte[] NAMESPACE_FAMILY_INFO_BYTES = Bytes.toBytes(NAMESPACE_FAMILY_INFO);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* @deprecated namespace table has been folded into the ns family in meta table, do not use this
|
||||||
|
* any more.
|
||||||
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
|
@Deprecated
|
||||||
public final static byte[] NAMESPACE_COL_DESC_BYTES = Bytes.toBytes("d");
|
public final static byte[] NAMESPACE_COL_DESC_BYTES = Bytes.toBytes("d");
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
@ -245,22 +262,21 @@ public class TableDescriptorBuilder {
|
||||||
CP_HTD_ATTR_VALUE_PARAM_VALUE_PATTERN + "),?");
|
CP_HTD_ATTR_VALUE_PARAM_VALUE_PATTERN + "),?");
|
||||||
private static final Pattern CP_HTD_ATTR_KEY_PATTERN =
|
private static final Pattern CP_HTD_ATTR_KEY_PATTERN =
|
||||||
Pattern.compile("^coprocessor\\$([0-9]+)$", Pattern.CASE_INSENSITIVE);
|
Pattern.compile("^coprocessor\\$([0-9]+)$", Pattern.CASE_INSENSITIVE);
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Table descriptor for namespace table
|
* Table descriptor for namespace table
|
||||||
|
* @deprecated We have folded the data in namespace table into meta table, so do not use it any
|
||||||
|
* more.
|
||||||
*/
|
*/
|
||||||
// TODO We used to set CacheDataInL1 for NS table. When we have BucketCache in file mode, now the
|
@Deprecated
|
||||||
// NS data goes to File mode BC only. Test how that affect the system. If too much, we have to
|
public static final TableDescriptor NAMESPACE_TABLEDESC =
|
||||||
// rethink about adding back the setCacheDataInL1 for NS table.
|
TableDescriptorBuilder.newBuilder(TableName.NAMESPACE_TABLE_NAME)
|
||||||
public static final TableDescriptor NAMESPACE_TABLEDESC
|
|
||||||
= TableDescriptorBuilder.newBuilder(TableName.NAMESPACE_TABLE_NAME)
|
|
||||||
.setColumnFamily(ColumnFamilyDescriptorBuilder.newBuilder(NAMESPACE_FAMILY_INFO_BYTES)
|
.setColumnFamily(ColumnFamilyDescriptorBuilder.newBuilder(NAMESPACE_FAMILY_INFO_BYTES)
|
||||||
// Ten is arbitrary number. Keep versions to help debugging.
|
// Ten is arbitrary number. Keep versions to help debugging.
|
||||||
.setMaxVersions(10)
|
.setMaxVersions(10).setInMemory(true).setBlocksize(8 * 1024)
|
||||||
.setInMemory(true)
|
.setScope(HConstants.REPLICATION_SCOPE_LOCAL).build())
|
||||||
.setBlocksize(8 * 1024)
|
|
||||||
.setScope(HConstants.REPLICATION_SCOPE_LOCAL)
|
|
||||||
.build())
|
|
||||||
.build();
|
.build();
|
||||||
|
|
||||||
private final ModifyableTableDescriptor desc;
|
private final ModifyableTableDescriptor desc;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
|
|
@ -26,7 +26,6 @@ import java.util.Collections;
|
||||||
import java.util.List;
|
import java.util.List;
|
||||||
import java.util.UUID;
|
import java.util.UUID;
|
||||||
import java.util.regex.Pattern;
|
import java.util.regex.Pattern;
|
||||||
|
|
||||||
import org.apache.commons.lang3.ArrayUtils;
|
import org.apache.commons.lang3.ArrayUtils;
|
||||||
import org.apache.hadoop.hbase.util.Bytes;
|
import org.apache.hadoop.hbase.util.Bytes;
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
|
@ -508,6 +507,13 @@ public final class HConstants {
|
||||||
public static final byte[] REPLICATION_BARRIER_FAMILY =
|
public static final byte[] REPLICATION_BARRIER_FAMILY =
|
||||||
Bytes.toBytes(REPLICATION_BARRIER_FAMILY_STR);
|
Bytes.toBytes(REPLICATION_BARRIER_FAMILY_STR);
|
||||||
|
|
||||||
|
/** The namespace family as a string */
|
||||||
|
public static final String NAMESPACE_FAMILY_STR = "ns";
|
||||||
|
|
||||||
|
/** The namespace family */
|
||||||
|
public static final byte[] NAMESPACE_FAMILY = Bytes.toBytes(NAMESPACE_FAMILY_STR);
|
||||||
|
|
||||||
|
public static final byte[] NAMESPACE_COL_DESC_QUALIFIER = Bytes.toBytes("d");
|
||||||
/**
|
/**
|
||||||
* The meta table version column qualifier.
|
* The meta table version column qualifier.
|
||||||
* We keep current version of the meta table in this column in <code>-ROOT-</code>
|
* We keep current version of the meta table in this column in <code>-ROOT-</code>
|
||||||
|
|
|
@ -166,7 +166,7 @@ public class NamespaceDescriptor {
|
||||||
|
|
||||||
private Builder(NamespaceDescriptor ns) {
|
private Builder(NamespaceDescriptor ns) {
|
||||||
this.bName = ns.name;
|
this.bName = ns.name;
|
||||||
this.bConfiguration = ns.configuration;
|
this.bConfiguration.putAll(ns.configuration);
|
||||||
}
|
}
|
||||||
|
|
||||||
private Builder(String name) {
|
private Builder(String name) {
|
||||||
|
|
|
@ -80,9 +80,14 @@ public final class TableName implements Comparable<TableName> {
|
||||||
public static final TableName META_TABLE_NAME =
|
public static final TableName META_TABLE_NAME =
|
||||||
valueOf(NamespaceDescriptor.SYSTEM_NAMESPACE_NAME_STR, "meta");
|
valueOf(NamespaceDescriptor.SYSTEM_NAMESPACE_NAME_STR, "meta");
|
||||||
|
|
||||||
/** The Namespace table's name. */
|
/**
|
||||||
|
* The Namespace table's name.
|
||||||
|
* @deprecated We have folded the data in namespace table into meta table, so do not use it any
|
||||||
|
* more.
|
||||||
|
*/
|
||||||
|
@Deprecated
|
||||||
public static final TableName NAMESPACE_TABLE_NAME =
|
public static final TableName NAMESPACE_TABLE_NAME =
|
||||||
valueOf(NamespaceDescriptor.SYSTEM_NAMESPACE_NAME_STR, "namespace");
|
valueOf(NamespaceDescriptor.SYSTEM_NAMESPACE_NAME_STR, "namespace");
|
||||||
|
|
||||||
public static final String OLD_META_STR = ".META.";
|
public static final String OLD_META_STR = ".META.";
|
||||||
public static final String OLD_ROOT_STR = "-ROOT-";
|
public static final String OLD_ROOT_STR = "-ROOT-";
|
||||||
|
|
|
@ -117,7 +117,7 @@ enum CreateNamespaceState {
|
||||||
CREATE_NAMESPACE_PREPARE = 1;
|
CREATE_NAMESPACE_PREPARE = 1;
|
||||||
CREATE_NAMESPACE_CREATE_DIRECTORY = 2;
|
CREATE_NAMESPACE_CREATE_DIRECTORY = 2;
|
||||||
CREATE_NAMESPACE_INSERT_INTO_NS_TABLE = 3;
|
CREATE_NAMESPACE_INSERT_INTO_NS_TABLE = 3;
|
||||||
CREATE_NAMESPACE_UPDATE_ZK = 4;
|
CREATE_NAMESPACE_UPDATE_ZK = 4[deprecated=true];
|
||||||
CREATE_NAMESPACE_SET_NAMESPACE_QUOTA = 5;
|
CREATE_NAMESPACE_SET_NAMESPACE_QUOTA = 5;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -128,7 +128,7 @@ message CreateNamespaceStateData {
|
||||||
enum ModifyNamespaceState {
|
enum ModifyNamespaceState {
|
||||||
MODIFY_NAMESPACE_PREPARE = 1;
|
MODIFY_NAMESPACE_PREPARE = 1;
|
||||||
MODIFY_NAMESPACE_UPDATE_NS_TABLE = 2;
|
MODIFY_NAMESPACE_UPDATE_NS_TABLE = 2;
|
||||||
MODIFY_NAMESPACE_UPDATE_ZK = 3;
|
MODIFY_NAMESPACE_UPDATE_ZK = 3[deprecated=true];
|
||||||
}
|
}
|
||||||
|
|
||||||
message ModifyNamespaceStateData {
|
message ModifyNamespaceStateData {
|
||||||
|
@ -139,7 +139,7 @@ message ModifyNamespaceStateData {
|
||||||
enum DeleteNamespaceState {
|
enum DeleteNamespaceState {
|
||||||
DELETE_NAMESPACE_PREPARE = 1;
|
DELETE_NAMESPACE_PREPARE = 1;
|
||||||
DELETE_NAMESPACE_DELETE_FROM_NS_TABLE = 2;
|
DELETE_NAMESPACE_DELETE_FROM_NS_TABLE = 2;
|
||||||
DELETE_NAMESPACE_REMOVE_FROM_ZK = 3;
|
DELETE_NAMESPACE_REMOVE_FROM_ZK = 3[deprecated=true];
|
||||||
DELETE_NAMESPACE_DELETE_DIRECTORIES = 4;
|
DELETE_NAMESPACE_DELETE_DIRECTORIES = 4;
|
||||||
DELETE_NAMESPACE_REMOVE_NAMESPACE_QUOTA = 5;
|
DELETE_NAMESPACE_REMOVE_NAMESPACE_QUOTA = 5;
|
||||||
}
|
}
|
||||||
|
@ -475,7 +475,8 @@ message ReopenTableRegionsStateData {
|
||||||
}
|
}
|
||||||
|
|
||||||
enum InitMetaState {
|
enum InitMetaState {
|
||||||
INIT_META_ASSIGN_META = 1;
|
INIT_META_ASSIGN_META = 1;
|
||||||
|
INIT_META_CREATE_NAMESPACES = 2;
|
||||||
}
|
}
|
||||||
|
|
||||||
message InitMetaStateData {
|
message InitMetaStateData {
|
||||||
|
|
|
@ -19,7 +19,6 @@
|
||||||
package org.apache.hadoop.hbase.rsgroup;
|
package org.apache.hadoop.hbase.rsgroup;
|
||||||
|
|
||||||
import com.google.protobuf.ServiceException;
|
import com.google.protobuf.ServiceException;
|
||||||
|
|
||||||
import java.io.ByteArrayInputStream;
|
import java.io.ByteArrayInputStream;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import java.util.ArrayList;
|
import java.util.ArrayList;
|
||||||
|
@ -34,7 +33,6 @@ import java.util.Set;
|
||||||
import java.util.SortedSet;
|
import java.util.SortedSet;
|
||||||
import java.util.TreeSet;
|
import java.util.TreeSet;
|
||||||
import java.util.concurrent.atomic.AtomicBoolean;
|
import java.util.concurrent.atomic.AtomicBoolean;
|
||||||
|
|
||||||
import org.apache.hadoop.conf.Configuration;
|
import org.apache.hadoop.conf.Configuration;
|
||||||
import org.apache.hadoop.hbase.Cell;
|
import org.apache.hadoop.hbase.Cell;
|
||||||
import org.apache.hadoop.hbase.CellUtil;
|
import org.apache.hadoop.hbase.CellUtil;
|
||||||
|
@ -85,6 +83,7 @@ import org.slf4j.LoggerFactory;
|
||||||
import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
|
import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
|
||||||
import org.apache.hbase.thirdparty.com.google.common.collect.Maps;
|
import org.apache.hbase.thirdparty.com.google.common.collect.Maps;
|
||||||
import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
|
import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
|
import org.apache.hadoop.hbase.shaded.protobuf.RequestConverter;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos;
|
||||||
|
|
||||||
|
@ -757,12 +756,9 @@ final class RSGroupInfoManagerImpl implements RSGroupInfoManager {
|
||||||
assignedRegions.clear();
|
assignedRegions.clear();
|
||||||
found.set(true);
|
found.set(true);
|
||||||
try {
|
try {
|
||||||
conn.getTable(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
conn.getTable(RSGROUP_TABLE_NAME);
|
|
||||||
boolean rootMetaFound =
|
boolean rootMetaFound =
|
||||||
masterServices.getMetaTableLocator().verifyMetaRegionLocation(
|
masterServices.getMetaTableLocator().verifyMetaRegionLocation(
|
||||||
conn, masterServices.getZooKeeper(), 1);
|
conn, masterServices.getZooKeeper(), 1);
|
||||||
final AtomicBoolean nsFound = new AtomicBoolean(false);
|
|
||||||
if (rootMetaFound) {
|
if (rootMetaFound) {
|
||||||
MetaTableAccessor.Visitor visitor = new DefaultVisitorBase() {
|
MetaTableAccessor.Visitor visitor = new DefaultVisitorBase() {
|
||||||
@Override
|
@Override
|
||||||
|
@ -791,36 +787,13 @@ final class RSGroupInfoManagerImpl implements RSGroupInfoManager {
|
||||||
}
|
}
|
||||||
foundRegions.add(info);
|
foundRegions.add(info);
|
||||||
}
|
}
|
||||||
if (TableName.NAMESPACE_TABLE_NAME.equals(info.getTable())) {
|
|
||||||
Cell cell = row.getColumnLatestCell(HConstants.CATALOG_FAMILY,
|
|
||||||
HConstants.SERVER_QUALIFIER);
|
|
||||||
ServerName sn = null;
|
|
||||||
if(cell != null) {
|
|
||||||
sn = ServerName.parseVersionedServerName(CellUtil.cloneValue(cell));
|
|
||||||
}
|
|
||||||
if (sn == null) {
|
|
||||||
nsFound.set(false);
|
|
||||||
} else if (tsm.isTableState(TableName.NAMESPACE_TABLE_NAME,
|
|
||||||
TableState.State.ENABLED)) {
|
|
||||||
try {
|
|
||||||
ClientProtos.ClientService.BlockingInterface rs = conn.getClient(sn);
|
|
||||||
ClientProtos.GetRequest request =
|
|
||||||
RequestConverter.buildGetRequest(info.getRegionName(),
|
|
||||||
new Get(ROW_KEY));
|
|
||||||
rs.get(null, request);
|
|
||||||
nsFound.set(true);
|
|
||||||
} catch(Exception ex) {
|
|
||||||
LOG.debug("Caught exception while verifying group region", ex);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
return true;
|
return true;
|
||||||
}
|
}
|
||||||
};
|
};
|
||||||
MetaTableAccessor.fullScanRegions(conn, visitor);
|
MetaTableAccessor.fullScanRegions(conn, visitor);
|
||||||
// if no regions in meta then we have to create the table
|
// if no regions in meta then we have to create the table
|
||||||
if (foundRegions.size() < 1 && rootMetaFound && !createSent && nsFound.get()) {
|
if (foundRegions.size() < 1 && rootMetaFound && !createSent) {
|
||||||
createRSGroupTable();
|
createRSGroupTable();
|
||||||
createSent = true;
|
createSent = true;
|
||||||
}
|
}
|
||||||
|
|
|
@ -93,8 +93,8 @@ public class TestRSGroupsBasics extends TestRSGroupsBase {
|
||||||
assertEquals(4, defaultInfo.getServers().size());
|
assertEquals(4, defaultInfo.getServers().size());
|
||||||
// Assignment of root and meta regions.
|
// Assignment of root and meta regions.
|
||||||
int count = master.getAssignmentManager().getRegionStates().getRegionAssignments().size();
|
int count = master.getAssignmentManager().getRegionStates().getRegionAssignments().size();
|
||||||
//3 meta,namespace, group
|
// 2 meta, group
|
||||||
assertEquals(3, count);
|
assertEquals(2, count);
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
|
|
|
@ -447,8 +447,6 @@ AssignmentManager assignmentManager = master.getAssignmentManager();
|
||||||
description = "The hbase:acl table holds information about acl.";
|
description = "The hbase:acl table holds information about acl.";
|
||||||
} else if (tableName.equals(VisibilityConstants.LABELS_TABLE_NAME)){
|
} else if (tableName.equals(VisibilityConstants.LABELS_TABLE_NAME)){
|
||||||
description = "The hbase:labels table holds information about visibility labels.";
|
description = "The hbase:labels table holds information about visibility labels.";
|
||||||
} else if (tableName.equals(TableName.NAMESPACE_TABLE_NAME)){
|
|
||||||
description = "The hbase:namespace table holds information about namespaces.";
|
|
||||||
} else if (tableName.equals(QuotaUtil.QUOTA_TABLE_NAME)){
|
} else if (tableName.equals(QuotaUtil.QUOTA_TABLE_NAME)){
|
||||||
description = "The hbase:quota table holds quota information about number" +
|
description = "The hbase:quota table holds quota information about number" +
|
||||||
" or size of requests in a given time frame.";
|
" or size of requests in a given time frame.";
|
||||||
|
|
|
@ -1,215 +0,0 @@
|
||||||
/**
|
|
||||||
* Licensed to the Apache Software Foundation (ASF) under one
|
|
||||||
* or more contributor license agreements. See the NOTICE file
|
|
||||||
* distributed with this work for additional information
|
|
||||||
* regarding copyright ownership. The ASF licenses this file
|
|
||||||
* to you under the Apache License, Version 2.0 (the
|
|
||||||
* "License"); you may not use this file except in compliance
|
|
||||||
* with the License. You may obtain a copy of the License at
|
|
||||||
*
|
|
||||||
* http://www.apache.org/licenses/LICENSE-2.0
|
|
||||||
*
|
|
||||||
* Unless required by applicable law or agreed to in writing, software
|
|
||||||
* distributed under the License is distributed on an "AS IS" BASIS,
|
|
||||||
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
|
||||||
* See the License for the specific language governing permissions and
|
|
||||||
* limitations under the License.
|
|
||||||
*/
|
|
||||||
|
|
||||||
package org.apache.hadoop.hbase;
|
|
||||||
|
|
||||||
import java.io.IOException;
|
|
||||||
import java.util.List;
|
|
||||||
import java.util.NavigableMap;
|
|
||||||
import java.util.NavigableSet;
|
|
||||||
import java.util.concurrent.ConcurrentSkipListMap;
|
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.util.Bytes;
|
|
||||||
import org.apache.hadoop.hbase.zookeeper.ZKListener;
|
|
||||||
import org.apache.hadoop.hbase.zookeeper.ZKUtil;
|
|
||||||
import org.apache.hadoop.hbase.zookeeper.ZKWatcher;
|
|
||||||
import org.apache.hadoop.hbase.zookeeper.ZNodePaths;
|
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
|
||||||
import org.apache.zookeeper.KeeperException;
|
|
||||||
import org.slf4j.Logger;
|
|
||||||
import org.slf4j.LoggerFactory;
|
|
||||||
|
|
||||||
import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
|
|
||||||
|
|
||||||
/**
|
|
||||||
* Class servers two purposes:
|
|
||||||
*
|
|
||||||
* 1. Broadcast NamespaceDescriptor information via ZK
|
|
||||||
* (Done by the Master)
|
|
||||||
* 2. Consume broadcasted NamespaceDescriptor changes
|
|
||||||
* (Done by the RegionServers)
|
|
||||||
*
|
|
||||||
*/
|
|
||||||
@InterfaceAudience.Private
|
|
||||||
public class ZKNamespaceManager extends ZKListener {
|
|
||||||
private static final Logger LOG = LoggerFactory.getLogger(ZKNamespaceManager.class);
|
|
||||||
private final String nsZNode;
|
|
||||||
private final NavigableMap<String,NamespaceDescriptor> cache;
|
|
||||||
|
|
||||||
public ZKNamespaceManager(ZKWatcher zkw) throws IOException {
|
|
||||||
super(zkw);
|
|
||||||
nsZNode = zkw.getZNodePaths().namespaceZNode;
|
|
||||||
cache = new ConcurrentSkipListMap<>();
|
|
||||||
}
|
|
||||||
|
|
||||||
public void start() throws IOException {
|
|
||||||
watcher.registerListener(this);
|
|
||||||
try {
|
|
||||||
if (ZKUtil.watchAndCheckExists(watcher, nsZNode)) {
|
|
||||||
List<ZKUtil.NodeAndData> existing =
|
|
||||||
ZKUtil.getChildDataAndWatchForNewChildren(watcher, nsZNode);
|
|
||||||
if (existing != null) {
|
|
||||||
refreshNodes(existing);
|
|
||||||
}
|
|
||||||
} else {
|
|
||||||
ZKUtil.createWithParents(watcher, nsZNode);
|
|
||||||
}
|
|
||||||
} catch (KeeperException e) {
|
|
||||||
throw new IOException("Failed to initialize ZKNamespaceManager", e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
public void stop() throws IOException {
|
|
||||||
this.watcher.unregisterListener(this);
|
|
||||||
}
|
|
||||||
|
|
||||||
public NamespaceDescriptor get(String name) {
|
|
||||||
return cache.get(name);
|
|
||||||
}
|
|
||||||
|
|
||||||
public void update(NamespaceDescriptor ns) throws IOException {
|
|
||||||
writeNamespace(ns);
|
|
||||||
cache.put(ns.getName(), ns);
|
|
||||||
}
|
|
||||||
|
|
||||||
public void remove(String name) throws IOException {
|
|
||||||
deleteNamespace(name);
|
|
||||||
cache.remove(name);
|
|
||||||
}
|
|
||||||
|
|
||||||
public NavigableSet<NamespaceDescriptor> list() throws IOException {
|
|
||||||
NavigableSet<NamespaceDescriptor> ret =
|
|
||||||
Sets.newTreeSet(NamespaceDescriptor.NAMESPACE_DESCRIPTOR_COMPARATOR);
|
|
||||||
for(NamespaceDescriptor ns: cache.values()) {
|
|
||||||
ret.add(ns);
|
|
||||||
}
|
|
||||||
return ret;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void nodeCreated(String path) {
|
|
||||||
if (nsZNode.equals(path)) {
|
|
||||||
try {
|
|
||||||
List<ZKUtil.NodeAndData> nodes =
|
|
||||||
ZKUtil.getChildDataAndWatchForNewChildren(watcher, nsZNode);
|
|
||||||
refreshNodes(nodes);
|
|
||||||
} catch (KeeperException ke) {
|
|
||||||
String msg = "Error reading data from zookeeper";
|
|
||||||
LOG.error(msg, ke);
|
|
||||||
watcher.abort(msg, ke);
|
|
||||||
} catch (IOException e) {
|
|
||||||
String msg = "Error parsing data from zookeeper";
|
|
||||||
LOG.error(msg, e);
|
|
||||||
watcher.abort(msg, e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void nodeDeleted(String path) {
|
|
||||||
if (nsZNode.equals(ZKUtil.getParent(path))) {
|
|
||||||
String nsName = ZKUtil.getNodeName(path);
|
|
||||||
cache.remove(nsName);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void nodeDataChanged(String path) {
|
|
||||||
if (nsZNode.equals(ZKUtil.getParent(path))) {
|
|
||||||
try {
|
|
||||||
byte[] data = ZKUtil.getDataAndWatch(watcher, path);
|
|
||||||
NamespaceDescriptor ns =
|
|
||||||
ProtobufUtil.toNamespaceDescriptor(
|
|
||||||
HBaseProtos.NamespaceDescriptor.parseFrom(data));
|
|
||||||
cache.put(ns.getName(), ns);
|
|
||||||
} catch (KeeperException ke) {
|
|
||||||
String msg = "Error reading data from zookeeper for node "+path;
|
|
||||||
LOG.error(msg, ke);
|
|
||||||
// only option is to abort
|
|
||||||
watcher.abort(msg, ke);
|
|
||||||
} catch (IOException ioe) {
|
|
||||||
String msg = "Error deserializing namespace: "+path;
|
|
||||||
LOG.error(msg, ioe);
|
|
||||||
watcher.abort(msg, ioe);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void nodeChildrenChanged(String path) {
|
|
||||||
if (nsZNode.equals(path)) {
|
|
||||||
try {
|
|
||||||
List<ZKUtil.NodeAndData> nodes =
|
|
||||||
ZKUtil.getChildDataAndWatchForNewChildren(watcher, nsZNode);
|
|
||||||
refreshNodes(nodes);
|
|
||||||
} catch (KeeperException ke) {
|
|
||||||
LOG.error("Error reading data from zookeeper for path "+path, ke);
|
|
||||||
watcher.abort("ZooKeeper error get node children for path "+path, ke);
|
|
||||||
} catch (IOException e) {
|
|
||||||
LOG.error("Error deserializing namespace child from: "+path, e);
|
|
||||||
watcher.abort("Error deserializing namespace child from: " + path, e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private void deleteNamespace(String name) throws IOException {
|
|
||||||
String zNode = ZNodePaths.joinZNode(nsZNode, name);
|
|
||||||
try {
|
|
||||||
ZKUtil.deleteNode(watcher, zNode);
|
|
||||||
} catch (KeeperException e) {
|
|
||||||
if (e instanceof KeeperException.NoNodeException) {
|
|
||||||
// If the node does not exist, it could be already deleted. Continue without fail.
|
|
||||||
LOG.warn("The ZNode " + zNode + " for namespace " + name + " does not exist.");
|
|
||||||
} else {
|
|
||||||
LOG.error("Failed updating permissions for namespace " + name, e);
|
|
||||||
throw new IOException("Failed updating permissions for namespace " + name, e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private void writeNamespace(NamespaceDescriptor ns) throws IOException {
|
|
||||||
String zNode = ZNodePaths.joinZNode(nsZNode, ns.getName());
|
|
||||||
try {
|
|
||||||
ZKUtil.createWithParents(watcher, zNode);
|
|
||||||
ZKUtil.updateExistingNodeData(watcher, zNode,
|
|
||||||
ProtobufUtil.toProtoNamespaceDescriptor(ns).toByteArray(), -1);
|
|
||||||
} catch (KeeperException e) {
|
|
||||||
LOG.error("Failed updating permissions for namespace "+ns.getName(), e);
|
|
||||||
throw new IOException("Failed updating permissions for namespace "+ns.getName(), e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private void refreshNodes(List<ZKUtil.NodeAndData> nodes) throws IOException {
|
|
||||||
for (ZKUtil.NodeAndData n : nodes) {
|
|
||||||
if (n.isEmpty()) continue;
|
|
||||||
String path = n.getNode();
|
|
||||||
String namespace = ZKUtil.getNodeName(path);
|
|
||||||
byte[] nodeData = n.getData();
|
|
||||||
if (LOG.isTraceEnabled()) {
|
|
||||||
LOG.trace("Updating namespace cache from node " + namespace + " with data: " +
|
|
||||||
Bytes.toStringBinary(nodeData));
|
|
||||||
}
|
|
||||||
NamespaceDescriptor ns =
|
|
||||||
ProtobufUtil.toNamespaceDescriptor(
|
|
||||||
HBaseProtos.NamespaceDescriptor.parseFrom(nodeData));
|
|
||||||
cache.put(ns.getName(), ns);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
}
|
|
|
@ -18,15 +18,10 @@
|
||||||
package org.apache.hadoop.hbase.master;
|
package org.apache.hadoop.hbase.master;
|
||||||
|
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import java.util.ArrayList;
|
|
||||||
import java.util.Collections;
|
|
||||||
import java.util.List;
|
import java.util.List;
|
||||||
import java.util.Set;
|
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
||||||
import org.apache.hadoop.hbase.ServiceNotRunningException;
|
import org.apache.hadoop.hbase.ServiceNotRunningException;
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
|
||||||
import org.apache.hadoop.hbase.master.procedure.CreateNamespaceProcedure;
|
import org.apache.hadoop.hbase.master.procedure.CreateNamespaceProcedure;
|
||||||
import org.apache.hadoop.hbase.master.procedure.DeleteNamespaceProcedure;
|
import org.apache.hadoop.hbase.master.procedure.DeleteNamespaceProcedure;
|
||||||
import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
|
import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
|
||||||
|
@ -34,15 +29,17 @@ import org.apache.hadoop.hbase.master.procedure.ModifyNamespaceProcedure;
|
||||||
import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch;
|
import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch;
|
||||||
import org.apache.hadoop.hbase.procedure2.Procedure;
|
import org.apache.hadoop.hbase.procedure2.Procedure;
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
|
import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
|
||||||
import org.apache.hbase.thirdparty.com.google.common.util.concurrent.AbstractService;
|
|
||||||
import org.apache.hadoop.hbase.util.NonceKey;
|
import org.apache.hadoop.hbase.util.NonceKey;
|
||||||
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
|
|
||||||
|
import org.apache.hbase.thirdparty.com.google.common.collect.ImmutableList;
|
||||||
|
import org.apache.hbase.thirdparty.com.google.common.util.concurrent.AbstractService;
|
||||||
|
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
class ClusterSchemaServiceImpl extends AbstractService implements ClusterSchemaService {
|
class ClusterSchemaServiceImpl extends AbstractService implements ClusterSchemaService {
|
||||||
|
|
||||||
private final TableNamespaceManager tableNamespaceManager;
|
private final TableNamespaceManager tableNamespaceManager;
|
||||||
private final MasterServices masterServices;
|
private final MasterServices masterServices;
|
||||||
private final static List<NamespaceDescriptor> EMPTY_NAMESPACE_LIST =
|
|
||||||
Collections.unmodifiableList(new ArrayList<NamespaceDescriptor>(0));
|
|
||||||
|
|
||||||
ClusterSchemaServiceImpl(final MasterServices masterServices) {
|
ClusterSchemaServiceImpl(final MasterServices masterServices) {
|
||||||
this.masterServices = masterServices;
|
this.masterServices = masterServices;
|
||||||
|
@ -51,9 +48,10 @@ class ClusterSchemaServiceImpl extends AbstractService implements ClusterSchemaS
|
||||||
|
|
||||||
// All below are synchronized so consistent view on whether running or not.
|
// All below are synchronized so consistent view on whether running or not.
|
||||||
|
|
||||||
|
|
||||||
private synchronized void checkIsRunning() throws ServiceNotRunningException {
|
private synchronized void checkIsRunning() throws ServiceNotRunningException {
|
||||||
if (!isRunning()) throw new ServiceNotRunningException();
|
if (!isRunning()) {
|
||||||
|
throw new ServiceNotRunningException();
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
@ -70,10 +68,6 @@ class ClusterSchemaServiceImpl extends AbstractService implements ClusterSchemaS
|
||||||
protected void doStop() {
|
protected void doStop() {
|
||||||
// This is no stop for the table manager.
|
// This is no stop for the table manager.
|
||||||
notifyStopped();
|
notifyStopped();
|
||||||
TableNamespaceManager tnsm = getTableNamespaceManager();
|
|
||||||
if (tnsm != null) {
|
|
||||||
tnsm.stop("Stopping");
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
@ -123,10 +117,8 @@ class ClusterSchemaServiceImpl extends AbstractService implements ClusterSchemaS
|
||||||
@Override
|
@Override
|
||||||
public List<NamespaceDescriptor> getNamespaces() throws IOException {
|
public List<NamespaceDescriptor> getNamespaces() throws IOException {
|
||||||
checkIsRunning();
|
checkIsRunning();
|
||||||
Set<NamespaceDescriptor> set = getTableNamespaceManager().list();
|
return getTableNamespaceManager().list().stream()
|
||||||
if (set == null || set.isEmpty()) return EMPTY_NAMESPACE_LIST;
|
.sorted(NamespaceDescriptor.NAMESPACE_DESCRIPTOR_COMPARATOR)
|
||||||
List<NamespaceDescriptor> list = new ArrayList<>(set.size());
|
.collect(ImmutableList.toImmutableList());
|
||||||
list.addAll(set);
|
|
||||||
return Collections.unmodifiableList(list);
|
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -1112,10 +1112,7 @@ public class HMaster extends HRegionServer implements MasterServices {
|
||||||
getChoreService().scheduleChore(catalogJanitorChore);
|
getChoreService().scheduleChore(catalogJanitorChore);
|
||||||
this.serverManager.startChore();
|
this.serverManager.startChore();
|
||||||
|
|
||||||
// NAMESPACE READ!!!!
|
// Only for rolling upgrade, where we need to migrate the data in namespace table to meta table.
|
||||||
// Here we expect hbase:namespace to be online. See inside initClusterSchemaService.
|
|
||||||
// TODO: Fix this. Namespace is a pain being a sort-of system table. Fold it in to hbase:meta.
|
|
||||||
// isNamespace does like isMeta and waits until namespace is onlined before allowing progress.
|
|
||||||
if (!waitForNamespaceOnline()) {
|
if (!waitForNamespaceOnline()) {
|
||||||
return;
|
return;
|
||||||
}
|
}
|
||||||
|
@ -1243,20 +1240,28 @@ public class HMaster extends HRegionServer implements MasterServices {
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Check hbase:namespace table is assigned. If not, startup will hang looking for the ns table
|
* Check hbase:namespace table is assigned. If not, startup will hang looking for the ns table
|
||||||
* (TODO: Fix this! NS should not hold-up startup).
|
* <p/>
|
||||||
|
* This is for rolling upgrading, later we will migrate the data in ns table to the ns family of
|
||||||
|
* meta table. And if this is a new clsuter, this method will return immediately as there will be
|
||||||
|
* no namespace table/region.
|
||||||
* @return True if namespace table is up/online.
|
* @return True if namespace table is up/online.
|
||||||
*/
|
*/
|
||||||
@VisibleForTesting
|
private boolean waitForNamespaceOnline() throws InterruptedException, IOException {
|
||||||
public boolean waitForNamespaceOnline() throws InterruptedException {
|
TableState nsTableState =
|
||||||
List<RegionInfo> ris = this.assignmentManager.getRegionStates().
|
MetaTableAccessor.getTableState(getClusterConnection(), TableName.NAMESPACE_TABLE_NAME);
|
||||||
getRegionsOfTable(TableName.NAMESPACE_TABLE_NAME);
|
if (nsTableState == null || nsTableState.isDisabled()) {
|
||||||
|
// this means we have already migrated the data and disabled or deleted the namespace table,
|
||||||
|
// or this is a new depliy which does not have a namespace table from the beginning.
|
||||||
|
return true;
|
||||||
|
}
|
||||||
|
List<RegionInfo> ris =
|
||||||
|
this.assignmentManager.getRegionStates().getRegionsOfTable(TableName.NAMESPACE_TABLE_NAME);
|
||||||
if (ris.isEmpty()) {
|
if (ris.isEmpty()) {
|
||||||
// If empty, means we've not assigned the namespace table yet... Just return true so startup
|
// maybe this will not happen any more, but anyway, no harm to add a check here...
|
||||||
// continues and the namespace table gets created.
|
|
||||||
return true;
|
return true;
|
||||||
}
|
}
|
||||||
// Else there are namespace regions up in meta. Ensure they are assigned before we go on.
|
// Else there are namespace regions up in meta. Ensure they are assigned before we go on.
|
||||||
for (RegionInfo ri: ris) {
|
for (RegionInfo ri : ris) {
|
||||||
isRegionOnline(ri);
|
isRegionOnline(ri);
|
||||||
}
|
}
|
||||||
return true;
|
return true;
|
||||||
|
|
|
@ -19,324 +19,159 @@
|
||||||
package org.apache.hadoop.hbase.master;
|
package org.apache.hadoop.hbase.master;
|
||||||
|
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import java.io.InterruptedIOException;
|
import java.util.List;
|
||||||
import java.util.NavigableSet;
|
import java.util.concurrent.ConcurrentHashMap;
|
||||||
|
import java.util.concurrent.ConcurrentMap;
|
||||||
|
import java.util.stream.Collectors;
|
||||||
import org.apache.commons.lang3.StringUtils;
|
import org.apache.commons.lang3.StringUtils;
|
||||||
import org.apache.hadoop.conf.Configuration;
|
|
||||||
import org.apache.hadoop.hbase.Cell;
|
import org.apache.hadoop.hbase.Cell;
|
||||||
import org.apache.hadoop.hbase.CellBuilderFactory;
|
|
||||||
import org.apache.hadoop.hbase.CellBuilderType;
|
|
||||||
import org.apache.hadoop.hbase.CellUtil;
|
import org.apache.hadoop.hbase.CellUtil;
|
||||||
import org.apache.hadoop.hbase.DoNotRetryIOException;
|
import org.apache.hadoop.hbase.DoNotRetryIOException;
|
||||||
import org.apache.hadoop.hbase.HTableDescriptor;
|
import org.apache.hadoop.hbase.HConstants;
|
||||||
import org.apache.hadoop.hbase.MetaTableAccessor;
|
import org.apache.hadoop.hbase.MetaTableAccessor;
|
||||||
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.Stoppable;
|
|
||||||
import org.apache.hadoop.hbase.TableName;
|
import org.apache.hadoop.hbase.TableName;
|
||||||
import org.apache.hadoop.hbase.ZKNamespaceManager;
|
import org.apache.hadoop.hbase.client.BufferedMutator;
|
||||||
|
import org.apache.hadoop.hbase.client.Connection;
|
||||||
import org.apache.hadoop.hbase.client.Delete;
|
import org.apache.hadoop.hbase.client.Delete;
|
||||||
import org.apache.hadoop.hbase.client.Get;
|
|
||||||
import org.apache.hadoop.hbase.client.Put;
|
import org.apache.hadoop.hbase.client.Put;
|
||||||
import org.apache.hadoop.hbase.client.Result;
|
import org.apache.hadoop.hbase.client.Result;
|
||||||
import org.apache.hadoop.hbase.client.ResultScanner;
|
import org.apache.hadoop.hbase.client.ResultScanner;
|
||||||
|
import org.apache.hadoop.hbase.client.Scan;
|
||||||
import org.apache.hadoop.hbase.client.Table;
|
import org.apache.hadoop.hbase.client.Table;
|
||||||
import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
|
import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
|
||||||
import org.apache.hadoop.hbase.client.TableState;
|
import org.apache.hadoop.hbase.client.TableState;
|
||||||
import org.apache.hadoop.hbase.constraint.ConstraintException;
|
import org.apache.hadoop.hbase.constraint.ConstraintException;
|
||||||
import org.apache.hadoop.hbase.exceptions.TimeoutIOException;
|
import org.apache.hadoop.hbase.master.procedure.DisableTableProcedure;
|
||||||
import org.apache.hadoop.hbase.master.procedure.MasterProcedureEnv;
|
import org.apache.hadoop.hbase.util.Bytes;
|
||||||
import org.apache.hadoop.hbase.master.procedure.ProcedurePrepareLatch;
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureExecutor;
|
|
||||||
import org.apache.hbase.thirdparty.com.google.common.collect.Sets;
|
import org.apache.hbase.thirdparty.com.google.protobuf.CodedInputStream;
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos;
|
||||||
import org.apache.hadoop.hbase.util.Bytes;
|
|
||||||
import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
|
|
||||||
import org.apache.hadoop.hbase.util.Threads;
|
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
|
||||||
import org.slf4j.Logger;
|
|
||||||
import org.slf4j.LoggerFactory;
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* This is a helper class used internally to manage the namespace metadata that is stored in
|
* This is a helper class used internally to manage the namespace metadata that is stored in the ns
|
||||||
* TableName.NAMESPACE_TABLE_NAME. It also mirrors updates to the ZK store by forwarding updates to
|
* family in meta table.
|
||||||
* {@link org.apache.hadoop.hbase.ZKNamespaceManager}.
|
|
||||||
*
|
|
||||||
* WARNING: Do not use. Go via the higher-level {@link ClusterSchema} API instead. This manager
|
|
||||||
* is likely to go aways anyways.
|
|
||||||
*/
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
@edu.umd.cs.findbugs.annotations.SuppressWarnings(value="IS2_INCONSISTENT_SYNC",
|
public class TableNamespaceManager {
|
||||||
justification="TODO: synchronize access on nsTable but it is done in tiers above and this " +
|
|
||||||
"class is going away/shrinking")
|
|
||||||
public class TableNamespaceManager implements Stoppable {
|
|
||||||
private static final Logger LOG = LoggerFactory.getLogger(TableNamespaceManager.class);
|
|
||||||
private volatile boolean stopped = false;
|
|
||||||
|
|
||||||
private Configuration conf;
|
|
||||||
private MasterServices masterServices;
|
|
||||||
private Table nsTable = null; // FindBugs: IS2_INCONSISTENT_SYNC TODO: Access is not synchronized
|
|
||||||
private ZKNamespaceManager zkNamespaceManager;
|
|
||||||
private boolean initialized;
|
|
||||||
|
|
||||||
public static final String KEY_MAX_REGIONS = "hbase.namespace.quota.maxregions";
|
public static final String KEY_MAX_REGIONS = "hbase.namespace.quota.maxregions";
|
||||||
public static final String KEY_MAX_TABLES = "hbase.namespace.quota.maxtables";
|
public static final String KEY_MAX_TABLES = "hbase.namespace.quota.maxtables";
|
||||||
static final String NS_INIT_TIMEOUT = "hbase.master.namespace.init.timeout";
|
static final String NS_INIT_TIMEOUT = "hbase.master.namespace.init.timeout";
|
||||||
static final int DEFAULT_NS_INIT_TIMEOUT = 300000;
|
static final int DEFAULT_NS_INIT_TIMEOUT = 300000;
|
||||||
|
|
||||||
|
private final ConcurrentMap<String, NamespaceDescriptor> cache = new ConcurrentHashMap<>();
|
||||||
|
|
||||||
|
private final MasterServices masterServices;
|
||||||
|
|
||||||
TableNamespaceManager(MasterServices masterServices) {
|
TableNamespaceManager(MasterServices masterServices) {
|
||||||
this.masterServices = masterServices;
|
this.masterServices = masterServices;
|
||||||
this.conf = masterServices.getConfiguration();
|
}
|
||||||
|
|
||||||
|
private void migrateNamespaceTable() throws IOException {
|
||||||
|
try (Table nsTable = masterServices.getConnection().getTable(TableName.NAMESPACE_TABLE_NAME);
|
||||||
|
ResultScanner scanner = nsTable.getScanner(
|
||||||
|
new Scan().addFamily(TableDescriptorBuilder.NAMESPACE_FAMILY_INFO_BYTES).readAllVersions());
|
||||||
|
BufferedMutator mutator =
|
||||||
|
masterServices.getConnection().getBufferedMutator(TableName.META_TABLE_NAME)) {
|
||||||
|
for (Result result;;) {
|
||||||
|
result = scanner.next();
|
||||||
|
if (result == null) {
|
||||||
|
break;
|
||||||
|
}
|
||||||
|
Put put = new Put(result.getRow());
|
||||||
|
result
|
||||||
|
.getColumnCells(TableDescriptorBuilder.NAMESPACE_FAMILY_INFO_BYTES,
|
||||||
|
TableDescriptorBuilder.NAMESPACE_COL_DESC_BYTES)
|
||||||
|
.forEach(c -> put.addColumn(HConstants.NAMESPACE_FAMILY,
|
||||||
|
HConstants.NAMESPACE_COL_DESC_QUALIFIER, c.getTimestamp(), CellUtil.cloneValue(c)));
|
||||||
|
mutator.mutate(put);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
// schedule a disable procedure instead of block waiting here, as when disabling a table we will
|
||||||
|
// wait until master is initialized, but we are part of the initialization...
|
||||||
|
masterServices.getMasterProcedureExecutor().submitProcedure(
|
||||||
|
new DisableTableProcedure(masterServices.getMasterProcedureExecutor().getEnvironment(),
|
||||||
|
TableName.NAMESPACE_TABLE_NAME, false));
|
||||||
|
}
|
||||||
|
|
||||||
|
private void loadNamespaceIntoCache() throws IOException {
|
||||||
|
try (Table table = masterServices.getConnection().getTable(TableName.META_TABLE_NAME);
|
||||||
|
ResultScanner scanner = table.getScanner(HConstants.NAMESPACE_FAMILY)) {
|
||||||
|
for (Result result;;) {
|
||||||
|
result = scanner.next();
|
||||||
|
if (result == null) {
|
||||||
|
break;
|
||||||
|
}
|
||||||
|
Cell cell = result.getColumnLatestCell(HConstants.NAMESPACE_FAMILY,
|
||||||
|
HConstants.NAMESPACE_COL_DESC_QUALIFIER);
|
||||||
|
NamespaceDescriptor ns = ProtobufUtil
|
||||||
|
.toNamespaceDescriptor(HBaseProtos.NamespaceDescriptor.parseFrom(CodedInputStream
|
||||||
|
.newInstance(cell.getValueArray(), cell.getValueOffset(), cell.getValueLength())));
|
||||||
|
cache.put(ns.getName(), ns);
|
||||||
|
}
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
public void start() throws IOException {
|
public void start() throws IOException {
|
||||||
if (!MetaTableAccessor.tableExists(masterServices.getConnection(),
|
TableState nsTableState = MetaTableAccessor.getTableState(masterServices.getConnection(),
|
||||||
TableName.NAMESPACE_TABLE_NAME)) {
|
TableName.NAMESPACE_TABLE_NAME);
|
||||||
LOG.info("Namespace table not found. Creating...");
|
if (nsTableState != null && nsTableState.isEnabled()) {
|
||||||
createNamespaceTable(masterServices);
|
migrateNamespaceTable();
|
||||||
}
|
|
||||||
|
|
||||||
try {
|
|
||||||
// Wait for the namespace table to be initialized.
|
|
||||||
long startTime = EnvironmentEdgeManager.currentTime();
|
|
||||||
int timeout = conf.getInt(NS_INIT_TIMEOUT, DEFAULT_NS_INIT_TIMEOUT);
|
|
||||||
while (!isTableAvailableAndInitialized()) {
|
|
||||||
if (EnvironmentEdgeManager.currentTime() - startTime + 100 > timeout) {
|
|
||||||
// We can't do anything if ns is not online.
|
|
||||||
throw new IOException("Timedout " + timeout + "ms waiting for namespace table to "
|
|
||||||
+ "be assigned and enabled: " + getTableState());
|
|
||||||
}
|
|
||||||
Thread.sleep(100);
|
|
||||||
}
|
|
||||||
} catch (InterruptedException e) {
|
|
||||||
throw (InterruptedIOException) new InterruptedIOException().initCause(e);
|
|
||||||
}
|
}
|
||||||
|
loadNamespaceIntoCache();
|
||||||
}
|
}
|
||||||
|
|
||||||
private synchronized Table getNamespaceTable() throws IOException {
|
/**
|
||||||
if (!isTableNamespaceManagerInitialized()) {
|
|
||||||
throw new IOException(this.getClass().getName() + " isn't ready to serve");
|
|
||||||
}
|
|
||||||
return nsTable;
|
|
||||||
}
|
|
||||||
|
|
||||||
/*
|
|
||||||
* check whether a namespace has already existed.
|
* check whether a namespace has already existed.
|
||||||
*/
|
*/
|
||||||
public boolean doesNamespaceExist(final String namespaceName) throws IOException {
|
public boolean doesNamespaceExist(String namespaceName) throws IOException {
|
||||||
if (nsTable == null) {
|
return cache.containsKey(namespaceName);
|
||||||
throw new IOException(this.getClass().getName() + " isn't ready to serve");
|
|
||||||
}
|
|
||||||
return (get(nsTable, namespaceName) != null);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public synchronized NamespaceDescriptor get(String name) throws IOException {
|
public NamespaceDescriptor get(String name) throws IOException {
|
||||||
if (!isTableNamespaceManagerInitialized()) {
|
return cache.get(name);
|
||||||
return null;
|
|
||||||
}
|
|
||||||
return zkNamespaceManager.get(name);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
private NamespaceDescriptor get(Table table, String name) throws IOException {
|
public void addOrUpdateNamespace(NamespaceDescriptor ns) throws IOException {
|
||||||
Result res = table.get(new Get(Bytes.toBytes(name)));
|
insertNamespaceToMeta(masterServices.getConnection(), ns);
|
||||||
if (res.isEmpty()) {
|
cache.put(ns.getName(), ns);
|
||||||
return null;
|
|
||||||
}
|
|
||||||
byte[] val = CellUtil.cloneValue(res.getColumnLatestCell(
|
|
||||||
HTableDescriptor.NAMESPACE_FAMILY_INFO_BYTES, HTableDescriptor.NAMESPACE_COL_DESC_BYTES));
|
|
||||||
return
|
|
||||||
ProtobufUtil.toNamespaceDescriptor(
|
|
||||||
HBaseProtos.NamespaceDescriptor.parseFrom(val));
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public void insertIntoNSTable(final NamespaceDescriptor ns) throws IOException {
|
public static void insertNamespaceToMeta(Connection conn, NamespaceDescriptor ns)
|
||||||
if (nsTable == null) {
|
|
||||||
throw new IOException(this.getClass().getName() + " isn't ready to serve");
|
|
||||||
}
|
|
||||||
byte[] row = Bytes.toBytes(ns.getName());
|
|
||||||
Put p = new Put(row, true);
|
|
||||||
p.add(CellBuilderFactory.create(CellBuilderType.SHALLOW_COPY)
|
|
||||||
.setRow(row)
|
|
||||||
.setFamily(TableDescriptorBuilder.NAMESPACE_FAMILY_INFO_BYTES)
|
|
||||||
.setQualifier(TableDescriptorBuilder.NAMESPACE_COL_DESC_BYTES)
|
|
||||||
.setTimestamp(p.getTimestamp())
|
|
||||||
.setType(Cell.Type.Put)
|
|
||||||
.setValue(ProtobufUtil.toProtoNamespaceDescriptor(ns).toByteArray())
|
|
||||||
.build());
|
|
||||||
nsTable.put(p);
|
|
||||||
}
|
|
||||||
|
|
||||||
public void updateZKNamespaceManager(final NamespaceDescriptor ns) throws IOException {
|
|
||||||
try {
|
|
||||||
zkNamespaceManager.update(ns);
|
|
||||||
} catch (IOException ex) {
|
|
||||||
String msg = "Failed to update namespace information in ZK.";
|
|
||||||
LOG.error(msg, ex);
|
|
||||||
throw new IOException(msg, ex);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
public void removeFromNSTable(final String namespaceName) throws IOException {
|
|
||||||
if (nsTable == null) {
|
|
||||||
throw new IOException(this.getClass().getName() + " isn't ready to serve");
|
|
||||||
}
|
|
||||||
Delete d = new Delete(Bytes.toBytes(namespaceName));
|
|
||||||
nsTable.delete(d);
|
|
||||||
}
|
|
||||||
|
|
||||||
public void removeFromZKNamespaceManager(final String namespaceName) throws IOException {
|
|
||||||
zkNamespaceManager.remove(namespaceName);
|
|
||||||
}
|
|
||||||
|
|
||||||
public synchronized NavigableSet<NamespaceDescriptor> list() throws IOException {
|
|
||||||
NavigableSet<NamespaceDescriptor> ret =
|
|
||||||
Sets.newTreeSet(NamespaceDescriptor.NAMESPACE_DESCRIPTOR_COMPARATOR);
|
|
||||||
ResultScanner scanner =
|
|
||||||
getNamespaceTable().getScanner(HTableDescriptor.NAMESPACE_FAMILY_INFO_BYTES);
|
|
||||||
try {
|
|
||||||
for(Result r : scanner) {
|
|
||||||
byte[] val = CellUtil.cloneValue(r.getColumnLatestCell(
|
|
||||||
HTableDescriptor.NAMESPACE_FAMILY_INFO_BYTES,
|
|
||||||
HTableDescriptor.NAMESPACE_COL_DESC_BYTES));
|
|
||||||
ret.add(ProtobufUtil.toNamespaceDescriptor(
|
|
||||||
HBaseProtos.NamespaceDescriptor.parseFrom(val)));
|
|
||||||
}
|
|
||||||
} finally {
|
|
||||||
scanner.close();
|
|
||||||
}
|
|
||||||
return ret;
|
|
||||||
}
|
|
||||||
|
|
||||||
private void createNamespaceTable(MasterServices masterServices) throws IOException {
|
|
||||||
masterServices.createSystemTable(HTableDescriptor.NAMESPACE_TABLEDESC);
|
|
||||||
}
|
|
||||||
|
|
||||||
@SuppressWarnings("deprecation")
|
|
||||||
private boolean isTableNamespaceManagerInitialized() throws IOException {
|
|
||||||
if (initialized) {
|
|
||||||
this.nsTable = this.masterServices.getConnection().getTable(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
return true;
|
|
||||||
}
|
|
||||||
return false;
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* Create Namespace in a blocking manner. Keeps trying until
|
|
||||||
* {@link ClusterSchema#HBASE_MASTER_CLUSTER_SCHEMA_OPERATION_TIMEOUT_KEY} expires.
|
|
||||||
* Note, by-passes notifying coprocessors and name checks. Use for system namespaces only.
|
|
||||||
*/
|
|
||||||
private void blockingCreateNamespace(final NamespaceDescriptor namespaceDescriptor)
|
|
||||||
throws IOException {
|
throws IOException {
|
||||||
ClusterSchema clusterSchema = this.masterServices.getClusterSchema();
|
byte[] row = Bytes.toBytes(ns.getName());
|
||||||
long procId = clusterSchema.createNamespace(namespaceDescriptor, null, ProcedurePrepareLatch.getNoopLatch());
|
Put put = new Put(row, true).addColumn(HConstants.NAMESPACE_FAMILY,
|
||||||
block(this.masterServices, procId);
|
HConstants.NAMESPACE_COL_DESC_QUALIFIER,
|
||||||
}
|
ProtobufUtil.toProtoNamespaceDescriptor(ns).toByteArray());
|
||||||
|
try (Table table = conn.getTable(TableName.META_TABLE_NAME)) {
|
||||||
|
table.put(put);
|
||||||
/**
|
|
||||||
* An ugly utility to be removed when refactor TableNamespaceManager.
|
|
||||||
* @throws TimeoutIOException
|
|
||||||
*/
|
|
||||||
private static void block(final MasterServices services, final long procId)
|
|
||||||
throws TimeoutIOException {
|
|
||||||
int timeoutInMillis = services.getConfiguration().
|
|
||||||
getInt(ClusterSchema.HBASE_MASTER_CLUSTER_SCHEMA_OPERATION_TIMEOUT_KEY,
|
|
||||||
ClusterSchema.DEFAULT_HBASE_MASTER_CLUSTER_SCHEMA_OPERATION_TIMEOUT);
|
|
||||||
long deadlineTs = EnvironmentEdgeManager.currentTime() + timeoutInMillis;
|
|
||||||
ProcedureExecutor<MasterProcedureEnv> procedureExecutor =
|
|
||||||
services.getMasterProcedureExecutor();
|
|
||||||
while(EnvironmentEdgeManager.currentTime() < deadlineTs) {
|
|
||||||
if (procedureExecutor.isFinished(procId)) return;
|
|
||||||
// Sleep some
|
|
||||||
Threads.sleep(10);
|
|
||||||
}
|
}
|
||||||
throw new TimeoutIOException("Procedure pid=" + procId + " is still running");
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
public void deleteNamespace(String namespaceName) throws IOException {
|
||||||
* This method checks if the namespace table is assigned and then
|
Delete d = new Delete(Bytes.toBytes(namespaceName));
|
||||||
* tries to create its Table reference. If it was already created before, it also makes
|
try (Table table = masterServices.getConnection().getTable(TableName.META_TABLE_NAME)) {
|
||||||
* sure that the connection isn't closed.
|
table.delete(d);
|
||||||
* @return true if the namespace table manager is ready to serve, false otherwise
|
|
||||||
*/
|
|
||||||
@SuppressWarnings("deprecation")
|
|
||||||
public synchronized boolean isTableAvailableAndInitialized()
|
|
||||||
throws IOException {
|
|
||||||
// Did we already get a table? If so, still make sure it's available
|
|
||||||
if (isTableNamespaceManagerInitialized()) {
|
|
||||||
return true;
|
|
||||||
}
|
}
|
||||||
|
cache.remove(namespaceName);
|
||||||
// Now check if the table is assigned, if not then fail fast
|
|
||||||
if (isTableAssigned() && isTableEnabled()) {
|
|
||||||
try {
|
|
||||||
boolean initGoodSofar = true;
|
|
||||||
nsTable = this.masterServices.getConnection().getTable(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
zkNamespaceManager = new ZKNamespaceManager(masterServices.getZooKeeper());
|
|
||||||
zkNamespaceManager.start();
|
|
||||||
|
|
||||||
if (get(nsTable, NamespaceDescriptor.DEFAULT_NAMESPACE.getName()) == null) {
|
|
||||||
blockingCreateNamespace(NamespaceDescriptor.DEFAULT_NAMESPACE);
|
|
||||||
}
|
|
||||||
if (get(nsTable, NamespaceDescriptor.SYSTEM_NAMESPACE.getName()) == null) {
|
|
||||||
blockingCreateNamespace(NamespaceDescriptor.SYSTEM_NAMESPACE);
|
|
||||||
}
|
|
||||||
|
|
||||||
if (!initGoodSofar) {
|
|
||||||
// some required namespace is created asynchronized. We should complete init later.
|
|
||||||
return false;
|
|
||||||
}
|
|
||||||
|
|
||||||
ResultScanner scanner = nsTable.getScanner(HTableDescriptor.NAMESPACE_FAMILY_INFO_BYTES);
|
|
||||||
try {
|
|
||||||
for (Result result : scanner) {
|
|
||||||
byte[] val = CellUtil.cloneValue(result.getColumnLatestCell(
|
|
||||||
HTableDescriptor.NAMESPACE_FAMILY_INFO_BYTES,
|
|
||||||
HTableDescriptor.NAMESPACE_COL_DESC_BYTES));
|
|
||||||
NamespaceDescriptor ns =
|
|
||||||
ProtobufUtil.toNamespaceDescriptor(
|
|
||||||
HBaseProtos.NamespaceDescriptor.parseFrom(val));
|
|
||||||
zkNamespaceManager.update(ns);
|
|
||||||
}
|
|
||||||
} finally {
|
|
||||||
scanner.close();
|
|
||||||
}
|
|
||||||
initialized = true;
|
|
||||||
return true;
|
|
||||||
} catch (IOException ie) {
|
|
||||||
LOG.warn("Caught exception in initializing namespace table manager", ie);
|
|
||||||
if (nsTable != null) {
|
|
||||||
nsTable.close();
|
|
||||||
}
|
|
||||||
throw ie;
|
|
||||||
}
|
|
||||||
}
|
|
||||||
return false;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
private TableState getTableState() throws IOException {
|
public List<NamespaceDescriptor> list() throws IOException {
|
||||||
return masterServices.getTableStateManager().getTableState(TableName.NAMESPACE_TABLE_NAME);
|
return cache.values().stream().collect(Collectors.toList());
|
||||||
}
|
|
||||||
|
|
||||||
private boolean isTableEnabled() throws IOException {
|
|
||||||
return getTableState().isEnabled();
|
|
||||||
}
|
|
||||||
|
|
||||||
private boolean isTableAssigned() {
|
|
||||||
// TODO: we have a better way now (wait on event)
|
|
||||||
return masterServices.getAssignmentManager()
|
|
||||||
.getRegionStates().hasTableRegionStates(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public void validateTableAndRegionCount(NamespaceDescriptor desc) throws IOException {
|
public void validateTableAndRegionCount(NamespaceDescriptor desc) throws IOException {
|
||||||
if (getMaxRegions(desc) <= 0) {
|
if (getMaxRegions(desc) <= 0) {
|
||||||
throw new ConstraintException("The max region quota for " + desc.getName()
|
throw new ConstraintException(
|
||||||
+ " is less than or equal to zero.");
|
"The max region quota for " + desc.getName() + " is less than or equal to zero.");
|
||||||
}
|
}
|
||||||
if (getMaxTables(desc) <= 0) {
|
if (getMaxTables(desc) <= 0) {
|
||||||
throw new ConstraintException("The max tables quota for " + desc.getName()
|
throw new ConstraintException(
|
||||||
+ " is less than or equal to zero.");
|
"The max tables quota for " + desc.getName() + " is less than or equal to zero.");
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -371,31 +206,4 @@ public class TableNamespaceManager implements Stoppable {
|
||||||
}
|
}
|
||||||
return maxRegions;
|
return maxRegions;
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
|
||||||
public boolean isStopped() {
|
|
||||||
return this.stopped;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
public void stop(String why) {
|
|
||||||
if (this.stopped) {
|
|
||||||
return;
|
|
||||||
}
|
|
||||||
try {
|
|
||||||
if (this.zkNamespaceManager != null) {
|
|
||||||
this.zkNamespaceManager.stop();
|
|
||||||
}
|
|
||||||
} catch (IOException ioe) {
|
|
||||||
LOG.warn("Failed NamespaceManager close", ioe);
|
|
||||||
}
|
|
||||||
try {
|
|
||||||
if (this.nsTable != null) {
|
|
||||||
this.nsTable.close();
|
|
||||||
}
|
|
||||||
} catch (IOException ioe) {
|
|
||||||
LOG.warn("Failed Namespace Table close", ioe);
|
|
||||||
}
|
|
||||||
this.stopped = true;
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
|
@ -366,7 +366,7 @@ public final class LockProcedure extends Procedure<MasterProcedureEnv>
|
||||||
}
|
}
|
||||||
|
|
||||||
private LockInterface setupNamespaceLock() throws IllegalArgumentException {
|
private LockInterface setupNamespaceLock() throws IllegalArgumentException {
|
||||||
this.tableName = TableName.NAMESPACE_TABLE_NAME;
|
this.tableName = TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME;
|
||||||
switch (type) {
|
switch (type) {
|
||||||
case EXCLUSIVE:
|
case EXCLUSIVE:
|
||||||
this.opType = TableOperationType.EDIT;
|
this.opType = TableOperationType.EDIT;
|
||||||
|
|
|
@ -18,18 +18,25 @@
|
||||||
|
|
||||||
package org.apache.hadoop.hbase.master.procedure;
|
package org.apache.hadoop.hbase.master.procedure;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.TableName;
|
import org.apache.hadoop.hbase.TableName;
|
||||||
|
import org.apache.hadoop.hbase.master.MasterFileSystem;
|
||||||
|
import org.apache.hadoop.hbase.master.TableNamespaceManager;
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
|
|
||||||
|
import org.apache.hbase.thirdparty.com.google.common.annotations.VisibleForTesting;
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.procedure2.StateMachineProcedure;
|
import org.apache.hadoop.hbase.procedure2.StateMachineProcedure;
|
||||||
|
import org.apache.hadoop.hbase.util.FSUtils;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Base class for all the Namespace procedures that want to use a StateMachineProcedure.
|
* Base class for all the Namespace procedures that want to use a StateMachineProcedure. It provide
|
||||||
* It provide some basic helpers like basic locking and basic toStringClassDetails().
|
* some basic helpers like basic locking and basic toStringClassDetails().
|
||||||
*/
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
public abstract class AbstractStateMachineNamespaceProcedure<TState>
|
public abstract class AbstractStateMachineNamespaceProcedure<TState>
|
||||||
extends StateMachineProcedure<MasterProcedureEnv, TState>
|
extends StateMachineProcedure<MasterProcedureEnv, TState> implements TableProcedureInterface {
|
||||||
implements TableProcedureInterface {
|
|
||||||
|
|
||||||
private final ProcedurePrepareLatch syncLatch;
|
private final ProcedurePrepareLatch syncLatch;
|
||||||
|
|
||||||
|
@ -52,7 +59,7 @@ public abstract class AbstractStateMachineNamespaceProcedure<TState>
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public TableName getTableName() {
|
public TableName getTableName() {
|
||||||
return TableName.NAMESPACE_TABLE_NAME;
|
return DUMMY_NAMESPACE_TABLE_NAME;
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
@ -83,6 +90,35 @@ public abstract class AbstractStateMachineNamespaceProcedure<TState>
|
||||||
env.getProcedureScheduler().wakeNamespaceExclusiveLock(this, getNamespaceName());
|
env.getProcedureScheduler().wakeNamespaceExclusiveLock(this, getNamespaceName());
|
||||||
}
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Insert/update the row into the ns family of meta table.
|
||||||
|
* @param env MasterProcedureEnv
|
||||||
|
*/
|
||||||
|
protected static void addOrUpdateNamespace(MasterProcedureEnv env, NamespaceDescriptor ns)
|
||||||
|
throws IOException {
|
||||||
|
getTableNamespaceManager(env).addOrUpdateNamespace(ns);
|
||||||
|
}
|
||||||
|
|
||||||
|
protected static TableNamespaceManager getTableNamespaceManager(MasterProcedureEnv env) {
|
||||||
|
return env.getMasterServices().getClusterSchema().getTableNamespaceManager();
|
||||||
|
}
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Create the namespace directory
|
||||||
|
* @param env MasterProcedureEnv
|
||||||
|
* @param nsDescriptor NamespaceDescriptor
|
||||||
|
*/
|
||||||
|
protected static void createDirectory(MasterProcedureEnv env, NamespaceDescriptor nsDescriptor)
|
||||||
|
throws IOException {
|
||||||
|
createDirectory(env.getMasterServices().getMasterFileSystem(), nsDescriptor);
|
||||||
|
}
|
||||||
|
|
||||||
|
@VisibleForTesting
|
||||||
|
public static void createDirectory(MasterFileSystem mfs, NamespaceDescriptor nsDescriptor)
|
||||||
|
throws IOException {
|
||||||
|
mfs.getFileSystem().mkdirs(FSUtils.getNamespaceDir(mfs.getRootDir(), nsDescriptor.getName()));
|
||||||
|
}
|
||||||
|
|
||||||
protected void releaseSyncLatch() {
|
protected void releaseSyncLatch() {
|
||||||
ProcedurePrepareLatch.releaseLatch(syncLatch, this);
|
ProcedurePrepareLatch.releaseLatch(syncLatch, this);
|
||||||
}
|
}
|
||||||
|
|
|
@ -15,23 +15,19 @@
|
||||||
* See the License for the specific language governing permissions and
|
* See the License for the specific language governing permissions and
|
||||||
* limitations under the License.
|
* limitations under the License.
|
||||||
*/
|
*/
|
||||||
|
|
||||||
package org.apache.hadoop.hbase.master.procedure;
|
package org.apache.hadoop.hbase.master.procedure;
|
||||||
|
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.NamespaceExistException;
|
import org.apache.hadoop.hbase.NamespaceExistException;
|
||||||
|
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
import org.slf4j.Logger;
|
import org.slf4j.Logger;
|
||||||
import org.slf4j.LoggerFactory;
|
import org.slf4j.LoggerFactory;
|
||||||
import org.apache.hadoop.hbase.master.MasterFileSystem;
|
|
||||||
import org.apache.hadoop.hbase.master.TableNamespaceManager;
|
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.CreateNamespaceState;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.CreateNamespaceState;
|
||||||
import org.apache.hadoop.hbase.util.FSUtils;
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* The procedure to create a new namespace.
|
* The procedure to create a new namespace.
|
||||||
|
@ -42,10 +38,8 @@ public class CreateNamespaceProcedure
|
||||||
private static final Logger LOG = LoggerFactory.getLogger(CreateNamespaceProcedure.class);
|
private static final Logger LOG = LoggerFactory.getLogger(CreateNamespaceProcedure.class);
|
||||||
|
|
||||||
private NamespaceDescriptor nsDescriptor;
|
private NamespaceDescriptor nsDescriptor;
|
||||||
private Boolean traceEnabled;
|
|
||||||
|
|
||||||
public CreateNamespaceProcedure() {
|
public CreateNamespaceProcedure() {
|
||||||
this.traceEnabled = null;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public CreateNamespaceProcedure(final MasterProcedureEnv env,
|
public CreateNamespaceProcedure(final MasterProcedureEnv env,
|
||||||
|
@ -57,43 +51,40 @@ public class CreateNamespaceProcedure
|
||||||
final NamespaceDescriptor nsDescriptor, ProcedurePrepareLatch latch) {
|
final NamespaceDescriptor nsDescriptor, ProcedurePrepareLatch latch) {
|
||||||
super(env, latch);
|
super(env, latch);
|
||||||
this.nsDescriptor = nsDescriptor;
|
this.nsDescriptor = nsDescriptor;
|
||||||
this.traceEnabled = null;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected Flow executeFromState(final MasterProcedureEnv env, final CreateNamespaceState state)
|
protected Flow executeFromState(final MasterProcedureEnv env, final CreateNamespaceState state)
|
||||||
throws InterruptedException {
|
throws InterruptedException {
|
||||||
if (isTraceEnabled()) {
|
LOG.trace("{} execute state={}", this, state);
|
||||||
LOG.trace(this + " execute state=" + state);
|
|
||||||
}
|
|
||||||
try {
|
try {
|
||||||
switch (state) {
|
switch (state) {
|
||||||
case CREATE_NAMESPACE_PREPARE:
|
case CREATE_NAMESPACE_PREPARE:
|
||||||
boolean success = prepareCreate(env);
|
boolean success = prepareCreate(env);
|
||||||
releaseSyncLatch();
|
releaseSyncLatch();
|
||||||
if (!success) {
|
if (!success) {
|
||||||
assert isFailed() : "createNamespace should have an exception here";
|
assert isFailed() : "createNamespace should have an exception here";
|
||||||
|
return Flow.NO_MORE_STATE;
|
||||||
|
}
|
||||||
|
setNextState(CreateNamespaceState.CREATE_NAMESPACE_CREATE_DIRECTORY);
|
||||||
|
break;
|
||||||
|
case CREATE_NAMESPACE_CREATE_DIRECTORY:
|
||||||
|
createDirectory(env, nsDescriptor);
|
||||||
|
setNextState(CreateNamespaceState.CREATE_NAMESPACE_INSERT_INTO_NS_TABLE);
|
||||||
|
break;
|
||||||
|
case CREATE_NAMESPACE_INSERT_INTO_NS_TABLE:
|
||||||
|
addOrUpdateNamespace(env, nsDescriptor);
|
||||||
|
setNextState(CreateNamespaceState.CREATE_NAMESPACE_SET_NAMESPACE_QUOTA);
|
||||||
|
break;
|
||||||
|
case CREATE_NAMESPACE_UPDATE_ZK:
|
||||||
|
// not used any more
|
||||||
|
setNextState(CreateNamespaceState.CREATE_NAMESPACE_SET_NAMESPACE_QUOTA);
|
||||||
|
break;
|
||||||
|
case CREATE_NAMESPACE_SET_NAMESPACE_QUOTA:
|
||||||
|
setNamespaceQuota(env, nsDescriptor);
|
||||||
return Flow.NO_MORE_STATE;
|
return Flow.NO_MORE_STATE;
|
||||||
}
|
default:
|
||||||
setNextState(CreateNamespaceState.CREATE_NAMESPACE_CREATE_DIRECTORY);
|
throw new UnsupportedOperationException(this + " unhandled state=" + state);
|
||||||
break;
|
|
||||||
case CREATE_NAMESPACE_CREATE_DIRECTORY:
|
|
||||||
createDirectory(env, nsDescriptor);
|
|
||||||
setNextState(CreateNamespaceState.CREATE_NAMESPACE_INSERT_INTO_NS_TABLE);
|
|
||||||
break;
|
|
||||||
case CREATE_NAMESPACE_INSERT_INTO_NS_TABLE:
|
|
||||||
insertIntoNSTable(env, nsDescriptor);
|
|
||||||
setNextState(CreateNamespaceState.CREATE_NAMESPACE_UPDATE_ZK);
|
|
||||||
break;
|
|
||||||
case CREATE_NAMESPACE_UPDATE_ZK:
|
|
||||||
updateZKNamespaceManager(env, nsDescriptor);
|
|
||||||
setNextState(CreateNamespaceState.CREATE_NAMESPACE_SET_NAMESPACE_QUOTA);
|
|
||||||
break;
|
|
||||||
case CREATE_NAMESPACE_SET_NAMESPACE_QUOTA:
|
|
||||||
setNamespaceQuota(env, nsDescriptor);
|
|
||||||
return Flow.NO_MORE_STATE;
|
|
||||||
default:
|
|
||||||
throw new UnsupportedOperationException(this + " unhandled state=" + state);
|
|
||||||
}
|
}
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
if (isRollbackSupported(state)) {
|
if (isRollbackSupported(state)) {
|
||||||
|
@ -145,39 +136,26 @@ public class CreateNamespaceProcedure
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void serializeStateData(ProcedureStateSerializer serializer)
|
protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException {
|
||||||
throws IOException {
|
|
||||||
super.serializeStateData(serializer);
|
super.serializeStateData(serializer);
|
||||||
|
|
||||||
MasterProcedureProtos.CreateNamespaceStateData.Builder createNamespaceMsg =
|
MasterProcedureProtos.CreateNamespaceStateData.Builder createNamespaceMsg =
|
||||||
MasterProcedureProtos.CreateNamespaceStateData.newBuilder().setNamespaceDescriptor(
|
MasterProcedureProtos.CreateNamespaceStateData.newBuilder()
|
||||||
ProtobufUtil.toProtoNamespaceDescriptor(this.nsDescriptor));
|
.setNamespaceDescriptor(ProtobufUtil.toProtoNamespaceDescriptor(this.nsDescriptor));
|
||||||
serializer.serialize(createNamespaceMsg.build());
|
serializer.serialize(createNamespaceMsg.build());
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void deserializeStateData(ProcedureStateSerializer serializer)
|
protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException {
|
||||||
throws IOException {
|
|
||||||
super.deserializeStateData(serializer);
|
super.deserializeStateData(serializer);
|
||||||
|
|
||||||
MasterProcedureProtos.CreateNamespaceStateData createNamespaceMsg =
|
MasterProcedureProtos.CreateNamespaceStateData createNamespaceMsg =
|
||||||
serializer.deserialize(MasterProcedureProtos.CreateNamespaceStateData.class);
|
serializer.deserialize(MasterProcedureProtos.CreateNamespaceStateData.class);
|
||||||
nsDescriptor = ProtobufUtil.toNamespaceDescriptor(createNamespaceMsg.getNamespaceDescriptor());
|
nsDescriptor = ProtobufUtil.toNamespaceDescriptor(createNamespaceMsg.getNamespaceDescriptor());
|
||||||
}
|
}
|
||||||
|
|
||||||
private boolean isBootstrapNamespace() {
|
|
||||||
return nsDescriptor.equals(NamespaceDescriptor.DEFAULT_NAMESPACE) ||
|
|
||||||
nsDescriptor.equals(NamespaceDescriptor.SYSTEM_NAMESPACE);
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected boolean waitInitialized(MasterProcedureEnv env) {
|
protected boolean waitInitialized(MasterProcedureEnv env) {
|
||||||
// Namespace manager might not be ready if master is not fully initialized,
|
|
||||||
// return false to reject user namespace creation; return true for default
|
|
||||||
// and system namespace creation (this is part of master initialization).
|
|
||||||
if (isBootstrapNamespace()) {
|
|
||||||
return false;
|
|
||||||
}
|
|
||||||
return env.waitInitialized(this);
|
return env.waitInitialized(this);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -202,90 +180,26 @@ public class CreateNamespaceProcedure
|
||||||
/**
|
/**
|
||||||
* Action before any real action of creating namespace.
|
* Action before any real action of creating namespace.
|
||||||
* @param env MasterProcedureEnv
|
* @param env MasterProcedureEnv
|
||||||
* @throws IOException
|
|
||||||
*/
|
*/
|
||||||
private boolean prepareCreate(final MasterProcedureEnv env) throws IOException {
|
private boolean prepareCreate(final MasterProcedureEnv env) throws IOException {
|
||||||
if (getTableNamespaceManager(env).doesNamespaceExist(nsDescriptor.getName())) {
|
if (getTableNamespaceManager(env).doesNamespaceExist(nsDescriptor.getName())) {
|
||||||
setFailure("master-create-namespace",
|
setFailure("master-create-namespace",
|
||||||
new NamespaceExistException("Namespace " + nsDescriptor.getName() + " already exists"));
|
new NamespaceExistException("Namespace " + nsDescriptor.getName() + " already exists"));
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
getTableNamespaceManager(env).validateTableAndRegionCount(nsDescriptor);
|
getTableNamespaceManager(env).validateTableAndRegionCount(nsDescriptor);
|
||||||
return true;
|
return true;
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
|
||||||
* Create the namespace directory
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @param nsDescriptor NamespaceDescriptor
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
protected static void createDirectory(
|
|
||||||
final MasterProcedureEnv env,
|
|
||||||
final NamespaceDescriptor nsDescriptor) throws IOException {
|
|
||||||
MasterFileSystem mfs = env.getMasterServices().getMasterFileSystem();
|
|
||||||
mfs.getFileSystem().mkdirs(
|
|
||||||
FSUtils.getNamespaceDir(mfs.getRootDir(), nsDescriptor.getName()));
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* Insert the row into ns table
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @param nsDescriptor NamespaceDescriptor
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
protected static void insertIntoNSTable(
|
|
||||||
final MasterProcedureEnv env,
|
|
||||||
final NamespaceDescriptor nsDescriptor) throws IOException {
|
|
||||||
getTableNamespaceManager(env).insertIntoNSTable(nsDescriptor);
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* Update ZooKeeper.
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @param nsDescriptor NamespaceDescriptor
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
protected static void updateZKNamespaceManager(
|
|
||||||
final MasterProcedureEnv env,
|
|
||||||
final NamespaceDescriptor nsDescriptor) throws IOException {
|
|
||||||
getTableNamespaceManager(env).updateZKNamespaceManager(nsDescriptor);
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Set quota for the namespace
|
* Set quota for the namespace
|
||||||
* @param env MasterProcedureEnv
|
* @param env MasterProcedureEnv
|
||||||
* @param nsDescriptor NamespaceDescriptor
|
* @param nsDescriptor NamespaceDescriptor
|
||||||
* @throws IOException
|
|
||||||
**/
|
**/
|
||||||
protected static void setNamespaceQuota(
|
private static void setNamespaceQuota(final MasterProcedureEnv env,
|
||||||
final MasterProcedureEnv env,
|
|
||||||
final NamespaceDescriptor nsDescriptor) throws IOException {
|
final NamespaceDescriptor nsDescriptor) throws IOException {
|
||||||
if (env.getMasterServices().isInitialized()) {
|
if (env.getMasterServices().isInitialized()) {
|
||||||
env.getMasterServices().getMasterQuotaManager().setNamespaceQuota(nsDescriptor);
|
env.getMasterServices().getMasterQuotaManager().setNamespaceQuota(nsDescriptor);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
private static TableNamespaceManager getTableNamespaceManager(final MasterProcedureEnv env) {
|
|
||||||
return env.getMasterServices().getClusterSchema().getTableNamespaceManager();
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* The procedure could be restarted from a different machine. If the variable is null, we need to
|
|
||||||
* retrieve it.
|
|
||||||
* @return traceEnabled
|
|
||||||
*/
|
|
||||||
private Boolean isTraceEnabled() {
|
|
||||||
if (traceEnabled == null) {
|
|
||||||
traceEnabled = LOG.isTraceEnabled();
|
|
||||||
}
|
|
||||||
return traceEnabled;
|
|
||||||
}
|
|
||||||
|
|
||||||
@Override
|
|
||||||
protected boolean shouldWaitClientAck(MasterProcedureEnv env) {
|
|
||||||
// hbase and default namespaces are created on bootstrap internally by the system
|
|
||||||
// the client does not know about this procedures.
|
|
||||||
return !isBootstrapNamespace();
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
|
@ -15,29 +15,27 @@
|
||||||
* See the License for the specific language governing permissions and
|
* See the License for the specific language governing permissions and
|
||||||
* limitations under the License.
|
* limitations under the License.
|
||||||
*/
|
*/
|
||||||
|
|
||||||
package org.apache.hadoop.hbase.master.procedure;
|
package org.apache.hadoop.hbase.master.procedure;
|
||||||
|
|
||||||
import java.io.FileNotFoundException;
|
import java.io.FileNotFoundException;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
|
|
||||||
import org.apache.hadoop.fs.FileStatus;
|
import org.apache.hadoop.fs.FileStatus;
|
||||||
import org.apache.hadoop.fs.FileSystem;
|
import org.apache.hadoop.fs.FileSystem;
|
||||||
import org.apache.hadoop.fs.Path;
|
import org.apache.hadoop.fs.Path;
|
||||||
import org.apache.hadoop.hbase.HConstants;
|
import org.apache.hadoop.hbase.HConstants;
|
||||||
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
||||||
|
import org.apache.hadoop.hbase.constraint.ConstraintException;
|
||||||
|
import org.apache.hadoop.hbase.master.MasterFileSystem;
|
||||||
|
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
||||||
|
import org.apache.hadoop.hbase.util.FSUtils;
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
import org.slf4j.Logger;
|
import org.slf4j.Logger;
|
||||||
import org.slf4j.LoggerFactory;
|
import org.slf4j.LoggerFactory;
|
||||||
import org.apache.hadoop.hbase.constraint.ConstraintException;
|
|
||||||
import org.apache.hadoop.hbase.master.MasterFileSystem;
|
|
||||||
import org.apache.hadoop.hbase.master.TableNamespaceManager;
|
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.DeleteNamespaceState;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.DeleteNamespaceState;
|
||||||
import org.apache.hadoop.hbase.util.FSUtils;
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* The procedure to remove a namespace.
|
* The procedure to remove a namespace.
|
||||||
|
@ -49,64 +47,59 @@ public class DeleteNamespaceProcedure
|
||||||
|
|
||||||
private NamespaceDescriptor nsDescriptor;
|
private NamespaceDescriptor nsDescriptor;
|
||||||
private String namespaceName;
|
private String namespaceName;
|
||||||
private Boolean traceEnabled;
|
|
||||||
|
|
||||||
public DeleteNamespaceProcedure() {
|
public DeleteNamespaceProcedure() {
|
||||||
this.nsDescriptor = null;
|
|
||||||
this.traceEnabled = null;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public DeleteNamespaceProcedure(final MasterProcedureEnv env, final String namespaceName) {
|
public DeleteNamespaceProcedure(MasterProcedureEnv env, String namespaceName) {
|
||||||
this(env, namespaceName, null);
|
this(env, namespaceName, null);
|
||||||
}
|
}
|
||||||
|
|
||||||
public DeleteNamespaceProcedure(final MasterProcedureEnv env, final String namespaceName,
|
public DeleteNamespaceProcedure(MasterProcedureEnv env, String namespaceName,
|
||||||
final ProcedurePrepareLatch latch) {
|
final ProcedurePrepareLatch latch) {
|
||||||
super(env, latch);
|
super(env, latch);
|
||||||
this.namespaceName = namespaceName;
|
this.namespaceName = namespaceName;
|
||||||
this.nsDescriptor = null;
|
|
||||||
this.traceEnabled = null;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected Flow executeFromState(final MasterProcedureEnv env, final DeleteNamespaceState state)
|
protected Flow executeFromState(MasterProcedureEnv env, DeleteNamespaceState state)
|
||||||
throws InterruptedException {
|
throws InterruptedException {
|
||||||
LOG.info(this.toString());
|
LOG.info(this.toString());
|
||||||
try {
|
try {
|
||||||
switch (state) {
|
switch (state) {
|
||||||
case DELETE_NAMESPACE_PREPARE:
|
case DELETE_NAMESPACE_PREPARE:
|
||||||
boolean present = prepareDelete(env);
|
boolean present = prepareDelete(env);
|
||||||
releaseSyncLatch();
|
releaseSyncLatch();
|
||||||
if (!present) {
|
if (!present) {
|
||||||
assert isFailed() : "Delete namespace should have an exception here";
|
assert isFailed() : "Delete namespace should have an exception here";
|
||||||
|
return Flow.NO_MORE_STATE;
|
||||||
|
}
|
||||||
|
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_DELETE_FROM_NS_TABLE);
|
||||||
|
break;
|
||||||
|
case DELETE_NAMESPACE_DELETE_FROM_NS_TABLE:
|
||||||
|
deleteNamespace(env, namespaceName);
|
||||||
|
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_DELETE_DIRECTORIES);
|
||||||
|
break;
|
||||||
|
case DELETE_NAMESPACE_REMOVE_FROM_ZK:
|
||||||
|
// not used any more
|
||||||
|
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_DELETE_DIRECTORIES);
|
||||||
|
break;
|
||||||
|
case DELETE_NAMESPACE_DELETE_DIRECTORIES:
|
||||||
|
deleteDirectory(env, namespaceName);
|
||||||
|
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_REMOVE_NAMESPACE_QUOTA);
|
||||||
|
break;
|
||||||
|
case DELETE_NAMESPACE_REMOVE_NAMESPACE_QUOTA:
|
||||||
|
removeNamespaceQuota(env, namespaceName);
|
||||||
return Flow.NO_MORE_STATE;
|
return Flow.NO_MORE_STATE;
|
||||||
}
|
default:
|
||||||
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_DELETE_FROM_NS_TABLE);
|
throw new UnsupportedOperationException(this + " unhandled state=" + state);
|
||||||
break;
|
|
||||||
case DELETE_NAMESPACE_DELETE_FROM_NS_TABLE:
|
|
||||||
deleteFromNSTable(env, namespaceName);
|
|
||||||
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_REMOVE_FROM_ZK);
|
|
||||||
break;
|
|
||||||
case DELETE_NAMESPACE_REMOVE_FROM_ZK:
|
|
||||||
removeFromZKNamespaceManager(env, namespaceName);
|
|
||||||
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_DELETE_DIRECTORIES);
|
|
||||||
break;
|
|
||||||
case DELETE_NAMESPACE_DELETE_DIRECTORIES:
|
|
||||||
deleteDirectory(env, namespaceName);
|
|
||||||
setNextState(DeleteNamespaceState.DELETE_NAMESPACE_REMOVE_NAMESPACE_QUOTA);
|
|
||||||
break;
|
|
||||||
case DELETE_NAMESPACE_REMOVE_NAMESPACE_QUOTA:
|
|
||||||
removeNamespaceQuota(env, namespaceName);
|
|
||||||
return Flow.NO_MORE_STATE;
|
|
||||||
default:
|
|
||||||
throw new UnsupportedOperationException(this + " unhandled state=" + state);
|
|
||||||
}
|
}
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
if (isRollbackSupported(state)) {
|
if (isRollbackSupported(state)) {
|
||||||
setFailure("master-delete-namespace", e);
|
setFailure("master-delete-namespace", e);
|
||||||
} else {
|
} else {
|
||||||
LOG.warn("Retriable error trying to delete namespace " + namespaceName +
|
LOG.warn("Retriable error trying to delete namespace " + namespaceName + " (in state=" +
|
||||||
" (in state=" + state + ")", e);
|
state + ")", e);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
return Flow.HAS_MORE_STATE;
|
return Flow.HAS_MORE_STATE;
|
||||||
|
@ -139,7 +132,7 @@ public class DeleteNamespaceProcedure
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected DeleteNamespaceState getState(final int stateId) {
|
protected DeleteNamespaceState getState(final int stateId) {
|
||||||
return DeleteNamespaceState.valueOf(stateId);
|
return DeleteNamespaceState.forNumber(stateId);
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
@ -153,30 +146,28 @@ public class DeleteNamespaceProcedure
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void serializeStateData(ProcedureStateSerializer serializer)
|
protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException {
|
||||||
throws IOException {
|
|
||||||
super.serializeStateData(serializer);
|
super.serializeStateData(serializer);
|
||||||
|
|
||||||
MasterProcedureProtos.DeleteNamespaceStateData.Builder deleteNamespaceMsg =
|
MasterProcedureProtos.DeleteNamespaceStateData.Builder deleteNamespaceMsg =
|
||||||
MasterProcedureProtos.DeleteNamespaceStateData.newBuilder().setNamespaceName(namespaceName);
|
MasterProcedureProtos.DeleteNamespaceStateData.newBuilder().setNamespaceName(namespaceName);
|
||||||
if (this.nsDescriptor != null) {
|
if (this.nsDescriptor != null) {
|
||||||
deleteNamespaceMsg.setNamespaceDescriptor(
|
deleteNamespaceMsg
|
||||||
ProtobufUtil.toProtoNamespaceDescriptor(this.nsDescriptor));
|
.setNamespaceDescriptor(ProtobufUtil.toProtoNamespaceDescriptor(this.nsDescriptor));
|
||||||
}
|
}
|
||||||
serializer.serialize(deleteNamespaceMsg.build());
|
serializer.serialize(deleteNamespaceMsg.build());
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void deserializeStateData(ProcedureStateSerializer serializer)
|
protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException {
|
||||||
throws IOException {
|
|
||||||
super.deserializeStateData(serializer);
|
super.deserializeStateData(serializer);
|
||||||
|
|
||||||
MasterProcedureProtos.DeleteNamespaceStateData deleteNamespaceMsg =
|
MasterProcedureProtos.DeleteNamespaceStateData deleteNamespaceMsg =
|
||||||
serializer.deserialize(MasterProcedureProtos.DeleteNamespaceStateData.class);
|
serializer.deserialize(MasterProcedureProtos.DeleteNamespaceStateData.class);
|
||||||
namespaceName = deleteNamespaceMsg.getNamespaceName();
|
namespaceName = deleteNamespaceMsg.getNamespaceName();
|
||||||
if (deleteNamespaceMsg.hasNamespaceDescriptor()) {
|
if (deleteNamespaceMsg.hasNamespaceDescriptor()) {
|
||||||
nsDescriptor =
|
nsDescriptor =
|
||||||
ProtobufUtil.toNamespaceDescriptor(deleteNamespaceMsg.getNamespaceDescriptor());
|
ProtobufUtil.toNamespaceDescriptor(deleteNamespaceMsg.getNamespaceDescriptor());
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -193,7 +184,6 @@ public class DeleteNamespaceProcedure
|
||||||
/**
|
/**
|
||||||
* Action before any real action of deleting namespace.
|
* Action before any real action of deleting namespace.
|
||||||
* @param env MasterProcedureEnv
|
* @param env MasterProcedureEnv
|
||||||
* @throws IOException
|
|
||||||
*/
|
*/
|
||||||
private boolean prepareDelete(final MasterProcedureEnv env) throws IOException {
|
private boolean prepareDelete(final MasterProcedureEnv env) throws IOException {
|
||||||
if (getTableNamespaceManager(env).doesNamespaceExist(namespaceName) == false) {
|
if (getTableNamespaceManager(env).doesNamespaceExist(namespaceName) == false) {
|
||||||
|
@ -201,8 +191,8 @@ public class DeleteNamespaceProcedure
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
if (NamespaceDescriptor.RESERVED_NAMESPACES.contains(namespaceName)) {
|
if (NamespaceDescriptor.RESERVED_NAMESPACES.contains(namespaceName)) {
|
||||||
setFailure("master-delete-namespace", new ConstraintException(
|
setFailure("master-delete-namespace",
|
||||||
"Reserved namespace "+ namespaceName +" cannot be removed."));
|
new ConstraintException("Reserved namespace " + namespaceName + " cannot be removed."));
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -214,9 +204,9 @@ public class DeleteNamespaceProcedure
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
if (tableCount > 0) {
|
if (tableCount > 0) {
|
||||||
setFailure("master-delete-namespace", new ConstraintException(
|
setFailure("master-delete-namespace",
|
||||||
"Only empty namespaces can be removed. Namespace "+ namespaceName + " has "
|
new ConstraintException("Only empty namespaces can be removed. Namespace " + namespaceName +
|
||||||
+ tableCount +" tables"));
|
" has " + tableCount + " tables"));
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -226,76 +216,28 @@ public class DeleteNamespaceProcedure
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* delete the row from namespace table
|
* delete the row from the ns family in meta table.
|
||||||
* @param env MasterProcedureEnv
|
* @param env MasterProcedureEnv
|
||||||
* @param namespaceName name of the namespace in string format
|
* @param namespaceName name of the namespace in string format
|
||||||
* @throws IOException
|
|
||||||
*/
|
*/
|
||||||
protected static void deleteFromNSTable(
|
private static void deleteNamespace(MasterProcedureEnv env, String namespaceName)
|
||||||
final MasterProcedureEnv env,
|
throws IOException {
|
||||||
final String namespaceName) throws IOException {
|
getTableNamespaceManager(env).deleteNamespace(namespaceName);
|
||||||
getTableNamespaceManager(env).removeFromNSTable(namespaceName);
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* undo the delete
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
private void undoDeleteFromNSTable(final MasterProcedureEnv env) {
|
|
||||||
try {
|
|
||||||
if (nsDescriptor != null) {
|
|
||||||
CreateNamespaceProcedure.insertIntoNSTable(env, nsDescriptor);
|
|
||||||
}
|
|
||||||
} catch (Exception e) {
|
|
||||||
// Ignore
|
|
||||||
LOG.debug("Rollback of deleteFromNSTable throws exception: " + e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* remove from ZooKeeper.
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @param namespaceName name of the namespace in string format
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
protected static void removeFromZKNamespaceManager(
|
|
||||||
final MasterProcedureEnv env,
|
|
||||||
final String namespaceName) throws IOException {
|
|
||||||
getTableNamespaceManager(env).removeFromZKNamespaceManager(namespaceName);
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* undo the remove from ZooKeeper
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
private void undoRemoveFromZKNamespaceManager(final MasterProcedureEnv env) {
|
|
||||||
try {
|
|
||||||
if (nsDescriptor != null) {
|
|
||||||
CreateNamespaceProcedure.updateZKNamespaceManager(env, nsDescriptor);
|
|
||||||
}
|
|
||||||
} catch (Exception e) {
|
|
||||||
// Ignore
|
|
||||||
LOG.debug("Rollback of removeFromZKNamespaceManager throws exception: " + e);
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Delete the namespace directories from the file system
|
* Delete the namespace directories from the file system
|
||||||
* @param env MasterProcedureEnv
|
* @param env MasterProcedureEnv
|
||||||
* @param namespaceName name of the namespace in string format
|
* @param namespaceName name of the namespace in string format
|
||||||
* @throws IOException
|
|
||||||
*/
|
*/
|
||||||
protected static void deleteDirectory(
|
private static void deleteDirectory(MasterProcedureEnv env, String namespaceName)
|
||||||
final MasterProcedureEnv env,
|
throws IOException {
|
||||||
final String namespaceName) throws IOException {
|
|
||||||
MasterFileSystem mfs = env.getMasterServices().getMasterFileSystem();
|
MasterFileSystem mfs = env.getMasterServices().getMasterFileSystem();
|
||||||
FileSystem fs = mfs.getFileSystem();
|
FileSystem fs = mfs.getFileSystem();
|
||||||
Path p = FSUtils.getNamespaceDir(mfs.getRootDir(), namespaceName);
|
Path p = FSUtils.getNamespaceDir(mfs.getRootDir(), namespaceName);
|
||||||
|
|
||||||
try {
|
try {
|
||||||
for(FileStatus status : fs.listStatus(p)) {
|
for (FileStatus status : fs.listStatus(p)) {
|
||||||
if (!HConstants.HBASE_NON_TABLE_DIRS.contains(status.getPath().getName())) {
|
if (!HConstants.HBASE_NON_TABLE_DIRS.contains(status.getPath().getName())) {
|
||||||
throw new IOException("Namespace directory contains table dir: " + status.getPath());
|
throw new IOException("Namespace directory contains table dir: " + status.getPath());
|
||||||
}
|
}
|
||||||
|
@ -309,58 +251,13 @@ public class DeleteNamespaceProcedure
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
|
||||||
* undo delete directory
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
private void rollbackDeleteDirectory(final MasterProcedureEnv env) throws IOException {
|
|
||||||
try {
|
|
||||||
CreateNamespaceProcedure.createDirectory(env, nsDescriptor);
|
|
||||||
} catch (Exception e) {
|
|
||||||
// Ignore exception
|
|
||||||
LOG.debug("Rollback of deleteDirectory throws exception: " + e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* remove quota for the namespace
|
* remove quota for the namespace
|
||||||
* @param env MasterProcedureEnv
|
* @param env MasterProcedureEnv
|
||||||
* @param namespaceName name of the namespace in string format
|
* @param namespaceName name of the namespace in string format
|
||||||
* @throws IOException
|
|
||||||
**/
|
**/
|
||||||
protected static void removeNamespaceQuota(
|
private static void removeNamespaceQuota(final MasterProcedureEnv env, final String namespaceName)
|
||||||
final MasterProcedureEnv env,
|
throws IOException {
|
||||||
final String namespaceName) throws IOException {
|
|
||||||
env.getMasterServices().getMasterQuotaManager().removeNamespaceQuota(namespaceName);
|
env.getMasterServices().getMasterQuotaManager().removeNamespaceQuota(namespaceName);
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
|
||||||
* undo remove quota for the namespace
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @throws IOException
|
|
||||||
**/
|
|
||||||
private void rollbacRemoveNamespaceQuota(final MasterProcedureEnv env) throws IOException {
|
|
||||||
try {
|
|
||||||
CreateNamespaceProcedure.setNamespaceQuota(env, nsDescriptor);
|
|
||||||
} catch (Exception e) {
|
|
||||||
// Ignore exception
|
|
||||||
LOG.debug("Rollback of removeNamespaceQuota throws exception: " + e);
|
|
||||||
}
|
|
||||||
}
|
|
||||||
|
|
||||||
private static TableNamespaceManager getTableNamespaceManager(final MasterProcedureEnv env) {
|
|
||||||
return env.getMasterServices().getClusterSchema().getTableNamespaceManager();
|
|
||||||
}
|
|
||||||
/**
|
|
||||||
* The procedure could be restarted from a different machine. If the variable is null, we need to
|
|
||||||
* retrieve it.
|
|
||||||
* @return traceEnabled
|
|
||||||
*/
|
|
||||||
private Boolean isTraceEnabled() {
|
|
||||||
if (traceEnabled == null) {
|
|
||||||
traceEnabled = LOG.isTraceEnabled();
|
|
||||||
}
|
|
||||||
return traceEnabled;
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
|
@ -17,6 +17,11 @@
|
||||||
*/
|
*/
|
||||||
package org.apache.hadoop.hbase.master.procedure;
|
package org.apache.hadoop.hbase.master.procedure;
|
||||||
|
|
||||||
|
import static org.apache.hadoop.hbase.NamespaceDescriptor.DEFAULT_NAMESPACE;
|
||||||
|
import static org.apache.hadoop.hbase.NamespaceDescriptor.SYSTEM_NAMESPACE;
|
||||||
|
import static org.apache.hadoop.hbase.master.TableNamespaceManager.insertNamespaceToMeta;
|
||||||
|
import static org.apache.hadoop.hbase.master.procedure.AbstractStateMachineNamespaceProcedure.createDirectory;
|
||||||
|
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import java.util.Arrays;
|
import java.util.Arrays;
|
||||||
import java.util.concurrent.CountDownLatch;
|
import java.util.concurrent.CountDownLatch;
|
||||||
|
@ -25,11 +30,15 @@ import org.apache.hadoop.hbase.client.RegionInfoBuilder;
|
||||||
import org.apache.hadoop.hbase.master.assignment.TransitRegionStateProcedure;
|
import org.apache.hadoop.hbase.master.assignment.TransitRegionStateProcedure;
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureSuspendedException;
|
import org.apache.hadoop.hbase.procedure2.ProcedureSuspendedException;
|
||||||
|
import org.apache.hadoop.hbase.procedure2.ProcedureUtil;
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureYieldException;
|
import org.apache.hadoop.hbase.procedure2.ProcedureYieldException;
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
|
import org.slf4j.Logger;
|
||||||
|
import org.slf4j.LoggerFactory;
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.InitMetaState;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.InitMetaState;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.InitMetaStateData;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.InitMetaStateData;
|
||||||
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.ProcedureProtos;
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* This procedure is used to initialize meta table for a new hbase deploy. It will just schedule an
|
* This procedure is used to initialize meta table for a new hbase deploy. It will just schedule an
|
||||||
|
@ -38,8 +47,12 @@ import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.I
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
public class InitMetaProcedure extends AbstractStateMachineTableProcedure<InitMetaState> {
|
public class InitMetaProcedure extends AbstractStateMachineTableProcedure<InitMetaState> {
|
||||||
|
|
||||||
|
private static final Logger LOG = LoggerFactory.getLogger(InitMetaProcedure.class);
|
||||||
|
|
||||||
private CountDownLatch latch = new CountDownLatch(1);
|
private CountDownLatch latch = new CountDownLatch(1);
|
||||||
|
|
||||||
|
private int attempts;
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public TableName getTableName() {
|
public TableName getTableName() {
|
||||||
return TableName.META_TABLE_NAME;
|
return TableName.META_TABLE_NAME;
|
||||||
|
@ -53,10 +66,32 @@ public class InitMetaProcedure extends AbstractStateMachineTableProcedure<InitMe
|
||||||
@Override
|
@Override
|
||||||
protected Flow executeFromState(MasterProcedureEnv env, InitMetaState state)
|
protected Flow executeFromState(MasterProcedureEnv env, InitMetaState state)
|
||||||
throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
|
throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException {
|
||||||
|
LOG.debug("Execute {}", this);
|
||||||
switch (state) {
|
switch (state) {
|
||||||
case INIT_META_ASSIGN_META:
|
case INIT_META_ASSIGN_META:
|
||||||
|
LOG.info("Going to assign meta");
|
||||||
addChildProcedure(env.getAssignmentManager()
|
addChildProcedure(env.getAssignmentManager()
|
||||||
.createAssignProcedures(Arrays.asList(RegionInfoBuilder.FIRST_META_REGIONINFO)));
|
.createAssignProcedures(Arrays.asList(RegionInfoBuilder.FIRST_META_REGIONINFO)));
|
||||||
|
setNextState(InitMetaState.INIT_META_CREATE_NAMESPACES);
|
||||||
|
return Flow.HAS_MORE_STATE;
|
||||||
|
case INIT_META_CREATE_NAMESPACES:
|
||||||
|
LOG.info("Going to create {} and {} namespaces", DEFAULT_NAMESPACE, SYSTEM_NAMESPACE);
|
||||||
|
try {
|
||||||
|
createDirectory(env, DEFAULT_NAMESPACE);
|
||||||
|
createDirectory(env, SYSTEM_NAMESPACE);
|
||||||
|
// here the TableNamespaceManager has not been initialized yet, so we have to insert the
|
||||||
|
// record directly into meta table, later the TableNamespaceManager will load these two
|
||||||
|
// namespaces when starting.
|
||||||
|
insertNamespaceToMeta(env.getMasterServices().getConnection(), DEFAULT_NAMESPACE);
|
||||||
|
insertNamespaceToMeta(env.getMasterServices().getConnection(), SYSTEM_NAMESPACE);
|
||||||
|
} catch (IOException e) {
|
||||||
|
long backoff = ProcedureUtil.getBackoffTimeMs(this.attempts++);
|
||||||
|
LOG.warn("Failed to init default and system namespaces, suspend {}secs", backoff, e);
|
||||||
|
setTimeout(Math.toIntExact(backoff));
|
||||||
|
setState(ProcedureProtos.ProcedureState.WAITING_TIMEOUT);
|
||||||
|
skipPersistence();
|
||||||
|
throw new ProcedureSuspendedException();
|
||||||
|
}
|
||||||
return Flow.NO_MORE_STATE;
|
return Flow.NO_MORE_STATE;
|
||||||
default:
|
default:
|
||||||
throw new UnsupportedOperationException("unhandled state=" + state);
|
throw new UnsupportedOperationException("unhandled state=" + state);
|
||||||
|
@ -69,6 +104,13 @@ public class InitMetaProcedure extends AbstractStateMachineTableProcedure<InitMe
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@Override
|
||||||
|
protected synchronized boolean setTimeoutFailure(MasterProcedureEnv env) {
|
||||||
|
setState(ProcedureProtos.ProcedureState.RUNNABLE);
|
||||||
|
env.getProcedureScheduler().addFront(this);
|
||||||
|
return false;
|
||||||
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected LockState acquireLock(MasterProcedureEnv env) {
|
protected LockState acquireLock(MasterProcedureEnv env) {
|
||||||
if (env.getProcedureScheduler().waitTableExclusiveLock(this, getTableName())) {
|
if (env.getProcedureScheduler().waitTableExclusiveLock(this, getTableName())) {
|
||||||
|
|
|
@ -808,11 +808,11 @@ public class MasterProcedureScheduler extends AbstractProcedureScheduler {
|
||||||
schedLock();
|
schedLock();
|
||||||
try {
|
try {
|
||||||
final LockAndQueue systemNamespaceTableLock =
|
final LockAndQueue systemNamespaceTableLock =
|
||||||
locking.getTableLock(TableName.NAMESPACE_TABLE_NAME);
|
locking.getTableLock(TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME);
|
||||||
if (!systemNamespaceTableLock.trySharedLock(procedure)) {
|
if (!systemNamespaceTableLock.trySharedLock(procedure)) {
|
||||||
waitProcedure(systemNamespaceTableLock, procedure);
|
waitProcedure(systemNamespaceTableLock, procedure);
|
||||||
logLockedResource(LockedResourceType.TABLE,
|
logLockedResource(LockedResourceType.TABLE,
|
||||||
TableName.NAMESPACE_TABLE_NAME.getNameAsString());
|
TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME.getNameAsString());
|
||||||
return true;
|
return true;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -840,13 +840,14 @@ public class MasterProcedureScheduler extends AbstractProcedureScheduler {
|
||||||
try {
|
try {
|
||||||
final LockAndQueue namespaceLock = locking.getNamespaceLock(namespace);
|
final LockAndQueue namespaceLock = locking.getNamespaceLock(namespace);
|
||||||
final LockAndQueue systemNamespaceTableLock =
|
final LockAndQueue systemNamespaceTableLock =
|
||||||
locking.getTableLock(TableName.NAMESPACE_TABLE_NAME);
|
locking.getTableLock(TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME);
|
||||||
int waitingCount = 0;
|
int waitingCount = 0;
|
||||||
if (namespaceLock.releaseExclusiveLock(procedure)) {
|
if (namespaceLock.releaseExclusiveLock(procedure)) {
|
||||||
waitingCount += wakeWaitingProcedures(namespaceLock);
|
waitingCount += wakeWaitingProcedures(namespaceLock);
|
||||||
}
|
}
|
||||||
if (systemNamespaceTableLock.releaseSharedLock()) {
|
if (systemNamespaceTableLock.releaseSharedLock()) {
|
||||||
addToRunQueue(tableRunQueue, getTableQueue(TableName.NAMESPACE_TABLE_NAME),
|
addToRunQueue(tableRunQueue,
|
||||||
|
getTableQueue(TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME),
|
||||||
() -> procedure + " released namespace exclusive lock");
|
() -> procedure + " released namespace exclusive lock");
|
||||||
waitingCount += wakeWaitingProcedures(systemNamespaceTableLock);
|
waitingCount += wakeWaitingProcedures(systemNamespaceTableLock);
|
||||||
}
|
}
|
||||||
|
|
|
@ -15,19 +15,17 @@
|
||||||
* See the License for the specific language governing permissions and
|
* See the License for the specific language governing permissions and
|
||||||
* limitations under the License.
|
* limitations under the License.
|
||||||
*/
|
*/
|
||||||
|
|
||||||
package org.apache.hadoop.hbase.master.procedure;
|
package org.apache.hadoop.hbase.master.procedure;
|
||||||
|
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
|
|
||||||
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
||||||
import org.apache.hadoop.hbase.constraint.ConstraintException;
|
import org.apache.hadoop.hbase.constraint.ConstraintException;
|
||||||
|
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
||||||
import org.apache.yetus.audience.InterfaceAudience;
|
import org.apache.yetus.audience.InterfaceAudience;
|
||||||
import org.slf4j.Logger;
|
import org.slf4j.Logger;
|
||||||
import org.slf4j.LoggerFactory;
|
import org.slf4j.LoggerFactory;
|
||||||
import org.apache.hadoop.hbase.master.TableNamespaceManager;
|
|
||||||
import org.apache.hadoop.hbase.procedure2.ProcedureStateSerializer;
|
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.ModifyNamespaceState;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.ModifyNamespaceState;
|
||||||
|
@ -42,11 +40,9 @@ public class ModifyNamespaceProcedure
|
||||||
|
|
||||||
private NamespaceDescriptor oldNsDescriptor;
|
private NamespaceDescriptor oldNsDescriptor;
|
||||||
private NamespaceDescriptor newNsDescriptor;
|
private NamespaceDescriptor newNsDescriptor;
|
||||||
private Boolean traceEnabled;
|
|
||||||
|
|
||||||
public ModifyNamespaceProcedure() {
|
public ModifyNamespaceProcedure() {
|
||||||
this.oldNsDescriptor = null;
|
this.oldNsDescriptor = null;
|
||||||
this.traceEnabled = null;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public ModifyNamespaceProcedure(final MasterProcedureEnv env,
|
public ModifyNamespaceProcedure(final MasterProcedureEnv env,
|
||||||
|
@ -59,36 +55,31 @@ public class ModifyNamespaceProcedure
|
||||||
super(env, latch);
|
super(env, latch);
|
||||||
this.oldNsDescriptor = null;
|
this.oldNsDescriptor = null;
|
||||||
this.newNsDescriptor = newNsDescriptor;
|
this.newNsDescriptor = newNsDescriptor;
|
||||||
this.traceEnabled = null;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected Flow executeFromState(final MasterProcedureEnv env, final ModifyNamespaceState state)
|
protected Flow executeFromState(final MasterProcedureEnv env, final ModifyNamespaceState state)
|
||||||
throws InterruptedException {
|
throws InterruptedException {
|
||||||
if (isTraceEnabled()) {
|
LOG.trace("{} execute state={}", this, state);
|
||||||
LOG.trace(this + " execute state=" + state);
|
|
||||||
}
|
|
||||||
|
|
||||||
try {
|
try {
|
||||||
switch (state) {
|
switch (state) {
|
||||||
case MODIFY_NAMESPACE_PREPARE:
|
case MODIFY_NAMESPACE_PREPARE:
|
||||||
boolean success = prepareModify(env);
|
boolean success = prepareModify(env);
|
||||||
releaseSyncLatch();
|
releaseSyncLatch();
|
||||||
if (!success) {
|
if (!success) {
|
||||||
assert isFailed() : "Modify namespace should have an exception here";
|
assert isFailed() : "Modify namespace should have an exception here";
|
||||||
|
return Flow.NO_MORE_STATE;
|
||||||
|
}
|
||||||
|
setNextState(ModifyNamespaceState.MODIFY_NAMESPACE_UPDATE_NS_TABLE);
|
||||||
|
break;
|
||||||
|
case MODIFY_NAMESPACE_UPDATE_NS_TABLE:
|
||||||
|
addOrUpdateNamespace(env, newNsDescriptor);
|
||||||
return Flow.NO_MORE_STATE;
|
return Flow.NO_MORE_STATE;
|
||||||
}
|
case MODIFY_NAMESPACE_UPDATE_ZK:
|
||||||
setNextState(ModifyNamespaceState.MODIFY_NAMESPACE_UPDATE_NS_TABLE);
|
// not used any more
|
||||||
break;
|
return Flow.NO_MORE_STATE;
|
||||||
case MODIFY_NAMESPACE_UPDATE_NS_TABLE:
|
default:
|
||||||
insertIntoNSTable(env);
|
throw new UnsupportedOperationException(this + " unhandled state=" + state);
|
||||||
setNextState(ModifyNamespaceState.MODIFY_NAMESPACE_UPDATE_ZK);
|
|
||||||
break;
|
|
||||||
case MODIFY_NAMESPACE_UPDATE_ZK:
|
|
||||||
updateZKNamespaceManager(env);
|
|
||||||
return Flow.NO_MORE_STATE;
|
|
||||||
default:
|
|
||||||
throw new UnsupportedOperationException(this + " unhandled state=" + state);
|
|
||||||
}
|
}
|
||||||
} catch (IOException e) {
|
} catch (IOException e) {
|
||||||
if (isRollbackSupported(state)) {
|
if (isRollbackSupported(state)) {
|
||||||
|
@ -116,7 +107,7 @@ public class ModifyNamespaceProcedure
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected boolean isRollbackSupported(final ModifyNamespaceState state) {
|
protected boolean isRollbackSupported(ModifyNamespaceState state) {
|
||||||
switch (state) {
|
switch (state) {
|
||||||
case MODIFY_NAMESPACE_PREPARE:
|
case MODIFY_NAMESPACE_PREPARE:
|
||||||
return true;
|
return true;
|
||||||
|
@ -126,12 +117,12 @@ public class ModifyNamespaceProcedure
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected ModifyNamespaceState getState(final int stateId) {
|
protected ModifyNamespaceState getState(int stateId) {
|
||||||
return ModifyNamespaceState.valueOf(stateId);
|
return ModifyNamespaceState.forNumber(stateId);
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected int getStateId(final ModifyNamespaceState state) {
|
protected int getStateId(ModifyNamespaceState state) {
|
||||||
return state.getNumber();
|
return state.getNumber();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -141,13 +132,12 @@ public class ModifyNamespaceProcedure
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void serializeStateData(ProcedureStateSerializer serializer)
|
protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException {
|
||||||
throws IOException {
|
|
||||||
super.serializeStateData(serializer);
|
super.serializeStateData(serializer);
|
||||||
|
|
||||||
MasterProcedureProtos.ModifyNamespaceStateData.Builder modifyNamespaceMsg =
|
MasterProcedureProtos.ModifyNamespaceStateData.Builder modifyNamespaceMsg =
|
||||||
MasterProcedureProtos.ModifyNamespaceStateData.newBuilder().setNamespaceDescriptor(
|
MasterProcedureProtos.ModifyNamespaceStateData.newBuilder()
|
||||||
ProtobufUtil.toProtoNamespaceDescriptor(this.newNsDescriptor));
|
.setNamespaceDescriptor(ProtobufUtil.toProtoNamespaceDescriptor(this.newNsDescriptor));
|
||||||
if (this.oldNsDescriptor != null) {
|
if (this.oldNsDescriptor != null) {
|
||||||
modifyNamespaceMsg.setUnmodifiedNamespaceDescriptor(
|
modifyNamespaceMsg.setUnmodifiedNamespaceDescriptor(
|
||||||
ProtobufUtil.toProtoNamespaceDescriptor(this.oldNsDescriptor));
|
ProtobufUtil.toProtoNamespaceDescriptor(this.oldNsDescriptor));
|
||||||
|
@ -156,17 +146,16 @@ public class ModifyNamespaceProcedure
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void deserializeStateData(ProcedureStateSerializer serializer)
|
protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException {
|
||||||
throws IOException {
|
|
||||||
super.deserializeStateData(serializer);
|
super.deserializeStateData(serializer);
|
||||||
|
|
||||||
MasterProcedureProtos.ModifyNamespaceStateData modifyNamespaceMsg =
|
MasterProcedureProtos.ModifyNamespaceStateData modifyNamespaceMsg =
|
||||||
serializer.deserialize(MasterProcedureProtos.ModifyNamespaceStateData.class);
|
serializer.deserialize(MasterProcedureProtos.ModifyNamespaceStateData.class);
|
||||||
newNsDescriptor =
|
newNsDescriptor =
|
||||||
ProtobufUtil.toNamespaceDescriptor(modifyNamespaceMsg.getNamespaceDescriptor());
|
ProtobufUtil.toNamespaceDescriptor(modifyNamespaceMsg.getNamespaceDescriptor());
|
||||||
if (modifyNamespaceMsg.hasUnmodifiedNamespaceDescriptor()) {
|
if (modifyNamespaceMsg.hasUnmodifiedNamespaceDescriptor()) {
|
||||||
oldNsDescriptor =
|
oldNsDescriptor =
|
||||||
ProtobufUtil.toNamespaceDescriptor(modifyNamespaceMsg.getUnmodifiedNamespaceDescriptor());
|
ProtobufUtil.toNamespaceDescriptor(modifyNamespaceMsg.getUnmodifiedNamespaceDescriptor());
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -187,8 +176,8 @@ public class ModifyNamespaceProcedure
|
||||||
*/
|
*/
|
||||||
private boolean prepareModify(final MasterProcedureEnv env) throws IOException {
|
private boolean prepareModify(final MasterProcedureEnv env) throws IOException {
|
||||||
if (getTableNamespaceManager(env).doesNamespaceExist(newNsDescriptor.getName()) == false) {
|
if (getTableNamespaceManager(env).doesNamespaceExist(newNsDescriptor.getName()) == false) {
|
||||||
setFailure("master-modify-namespace", new NamespaceNotFoundException(
|
setFailure("master-modify-namespace",
|
||||||
newNsDescriptor.getName()));
|
new NamespaceNotFoundException(newNsDescriptor.getName()));
|
||||||
return false;
|
return false;
|
||||||
}
|
}
|
||||||
try {
|
try {
|
||||||
|
@ -202,38 +191,4 @@ public class ModifyNamespaceProcedure
|
||||||
oldNsDescriptor = getTableNamespaceManager(env).get(newNsDescriptor.getName());
|
oldNsDescriptor = getTableNamespaceManager(env).get(newNsDescriptor.getName());
|
||||||
return true;
|
return true;
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
|
||||||
* Insert/update the row into namespace table
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
private void insertIntoNSTable(final MasterProcedureEnv env) throws IOException {
|
|
||||||
getTableNamespaceManager(env).insertIntoNSTable(newNsDescriptor);
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* Update ZooKeeper.
|
|
||||||
* @param env MasterProcedureEnv
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
|
||||||
private void updateZKNamespaceManager(final MasterProcedureEnv env) throws IOException {
|
|
||||||
getTableNamespaceManager(env).updateZKNamespaceManager(newNsDescriptor);
|
|
||||||
}
|
|
||||||
|
|
||||||
private TableNamespaceManager getTableNamespaceManager(final MasterProcedureEnv env) {
|
|
||||||
return env.getMasterServices().getClusterSchema().getTableNamespaceManager();
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* The procedure could be restarted from a different machine. If the variable is null, we need to
|
|
||||||
* retrieve it.
|
|
||||||
* @return traceEnabled
|
|
||||||
*/
|
|
||||||
private Boolean isTraceEnabled() {
|
|
||||||
if (traceEnabled == null) {
|
|
||||||
traceEnabled = LOG.isTraceEnabled();
|
|
||||||
}
|
|
||||||
return traceEnabled;
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
|
@ -26,6 +26,13 @@ import org.apache.yetus.audience.InterfaceAudience;
|
||||||
*/
|
*/
|
||||||
@InterfaceAudience.Private
|
@InterfaceAudience.Private
|
||||||
public interface TableProcedureInterface {
|
public interface TableProcedureInterface {
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Used for acquire/release lock for namespace related operations, just a place holder as we do
|
||||||
|
* not have namespace table any more.
|
||||||
|
*/
|
||||||
|
public static final TableName DUMMY_NAMESPACE_TABLE_NAME = TableName.NAMESPACE_TABLE_NAME;
|
||||||
|
|
||||||
public enum TableOperationType {
|
public enum TableOperationType {
|
||||||
CREATE, DELETE, DISABLE, EDIT, ENABLE, READ,
|
CREATE, DELETE, DISABLE, EDIT, ENABLE, READ,
|
||||||
REGION_EDIT, REGION_SPLIT, REGION_MERGE, REGION_ASSIGN, REGION_UNASSIGN,
|
REGION_EDIT, REGION_SPLIT, REGION_MERGE, REGION_ASSIGN, REGION_UNASSIGN,
|
||||||
|
|
|
@ -53,8 +53,8 @@ class TableQueue extends Queue<TableName> {
|
||||||
case ENABLE:
|
case ENABLE:
|
||||||
return true;
|
return true;
|
||||||
case EDIT:
|
case EDIT:
|
||||||
// we allow concurrent edit on the NS table
|
// we allow concurrent edit on the ns family in meta table
|
||||||
return !proc.getTableName().equals(TableName.NAMESPACE_TABLE_NAME);
|
return !proc.getTableName().equals(TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME);
|
||||||
case READ:
|
case READ:
|
||||||
return false;
|
return false;
|
||||||
// region operations are using the shared-lock on the table
|
// region operations are using the shared-lock on the table
|
||||||
|
|
|
@ -1,4 +1,4 @@
|
||||||
/*
|
/**
|
||||||
* Licensed to the Apache Software Foundation (ASF) under one
|
* Licensed to the Apache Software Foundation (ASF) under one
|
||||||
* or more contributor license agreements. See the NOTICE file
|
* or more contributor license agreements. See the NOTICE file
|
||||||
* distributed with this work for additional information
|
* distributed with this work for additional information
|
||||||
|
@ -95,7 +95,6 @@ import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.NotServingRegionException;
|
import org.apache.hadoop.hbase.NotServingRegionException;
|
||||||
import org.apache.hadoop.hbase.PrivateCellUtil;
|
import org.apache.hadoop.hbase.PrivateCellUtil;
|
||||||
import org.apache.hadoop.hbase.RegionTooBusyException;
|
import org.apache.hadoop.hbase.RegionTooBusyException;
|
||||||
import org.apache.hadoop.hbase.TableName;
|
|
||||||
import org.apache.hadoop.hbase.Tag;
|
import org.apache.hadoop.hbase.Tag;
|
||||||
import org.apache.hadoop.hbase.TagUtil;
|
import org.apache.hadoop.hbase.TagUtil;
|
||||||
import org.apache.hadoop.hbase.UnknownScannerException;
|
import org.apache.hadoop.hbase.UnknownScannerException;
|
||||||
|
@ -8298,8 +8297,7 @@ public class HRegion implements HeapSize, PropagatingConfigurationObserver, Regi
|
||||||
*/
|
*/
|
||||||
public byte[] checkSplit() {
|
public byte[] checkSplit() {
|
||||||
// Can't split META
|
// Can't split META
|
||||||
if (this.getRegionInfo().isMetaRegion() ||
|
if (this.getRegionInfo().isMetaRegion()) {
|
||||||
TableName.NAMESPACE_TABLE_NAME.equals(this.getRegionInfo().getTable())) {
|
|
||||||
if (shouldForceSplit()) {
|
if (shouldForceSplit()) {
|
||||||
LOG.warn("Cannot split meta region in HBase 0.20 and above");
|
LOG.warn("Cannot split meta region in HBase 0.20 and above");
|
||||||
}
|
}
|
||||||
|
|
|
@ -779,13 +779,10 @@ public class HRegionServer extends HasThread implements
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Create a 'smarter' Connection, one that is capable of by-passing RPC if the request is to
|
* Create a 'smarter' Connection, one that is capable of by-passing RPC if the request is to the
|
||||||
* the local server; i.e. a short-circuit Connection. Safe to use going to local or remote
|
* local server; i.e. a short-circuit Connection. Safe to use going to local or remote server.
|
||||||
* server. Create this instance in a method can be intercepted and mocked in tests.
|
|
||||||
* @throws IOException
|
|
||||||
*/
|
*/
|
||||||
@VisibleForTesting
|
private ClusterConnection createClusterConnection() throws IOException {
|
||||||
protected ClusterConnection createClusterConnection() throws IOException {
|
|
||||||
Configuration conf = this.conf;
|
Configuration conf = this.conf;
|
||||||
if (conf.get(HConstants.CLIENT_ZOOKEEPER_QUORUM) != null) {
|
if (conf.get(HConstants.CLIENT_ZOOKEEPER_QUORUM) != null) {
|
||||||
// Use server ZK cluster for server-issued connections, so we clone
|
// Use server ZK cluster for server-issued connections, so we clone
|
||||||
|
@ -796,8 +793,15 @@ public class HRegionServer extends HasThread implements
|
||||||
// Create a cluster connection that when appropriate, can short-circuit and go directly to the
|
// Create a cluster connection that when appropriate, can short-circuit and go directly to the
|
||||||
// local server if the request is to the local server bypassing RPC. Can be used for both local
|
// local server if the request is to the local server bypassing RPC. Can be used for both local
|
||||||
// and remote invocations.
|
// and remote invocations.
|
||||||
return ConnectionUtils.createShortCircuitConnection(conf, null, userProvider.getCurrent(),
|
ClusterConnection conn = ConnectionUtils.createShortCircuitConnection(conf, null,
|
||||||
serverName, rpcServices, rpcServices);
|
userProvider.getCurrent(), serverName, rpcServices, rpcServices);
|
||||||
|
// This is used to initialize the batch thread pool inside the connection implementation.
|
||||||
|
// When deploy a fresh cluster, we may first use the cluster connection in InitMetaProcedure,
|
||||||
|
// which will be executed inside the PEWorker, and then the batch thread pool will inherit the
|
||||||
|
// thread group of PEWorker, which will be destroy when shutting down the ProcedureExecutor. It
|
||||||
|
// will cause lots of procedure related UTs to fail, so here let's initialize it first, no harm.
|
||||||
|
conn.getTable(TableName.META_TABLE_NAME).close();
|
||||||
|
return conn;
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
@ -823,7 +827,6 @@ public class HRegionServer extends HasThread implements
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Setup our cluster connection if not already initialized.
|
* Setup our cluster connection if not already initialized.
|
||||||
* @throws IOException
|
|
||||||
*/
|
*/
|
||||||
protected synchronized void setupClusterConnection() throws IOException {
|
protected synchronized void setupClusterConnection() throws IOException {
|
||||||
if (clusterConnection == null) {
|
if (clusterConnection == null) {
|
||||||
|
|
|
@ -875,7 +875,7 @@ public class AccessController implements MasterCoprocessor, RegionCoprocessor,
|
||||||
// wait till the acl table is created.
|
// wait till the acl table is created.
|
||||||
if (AccessControlLists.isAclTable(desc)) {
|
if (AccessControlLists.isAclTable(desc)) {
|
||||||
this.aclTabAvailable = true;
|
this.aclTabAvailable = true;
|
||||||
} else if (!(TableName.NAMESPACE_TABLE_NAME.equals(desc.getTableName()))) {
|
} else {
|
||||||
if (!aclTabAvailable) {
|
if (!aclTabAvailable) {
|
||||||
LOG.warn("Not adding owner permission for table " + desc.getTableName() + ". "
|
LOG.warn("Not adding owner permission for table " + desc.getTableName() + ". "
|
||||||
+ AccessControlLists.ACL_TABLE_NAME + " is not yet created. "
|
+ AccessControlLists.ACL_TABLE_NAME + " is not yet created. "
|
||||||
|
|
|
@ -178,6 +178,17 @@ public class FSTableDescriptors implements TableDescriptors {
|
||||||
// Disable blooms for meta. Needs work. Seems to mess w/ getClosestOrBefore.
|
// Disable blooms for meta. Needs work. Seems to mess w/ getClosestOrBefore.
|
||||||
.setBloomFilterType(BloomType.NONE)
|
.setBloomFilterType(BloomType.NONE)
|
||||||
.build())
|
.build())
|
||||||
|
.setColumnFamily(ColumnFamilyDescriptorBuilder
|
||||||
|
.newBuilder(HConstants.NAMESPACE_FAMILY)
|
||||||
|
.setMaxVersions(conf.getInt(HConstants.HBASE_META_VERSIONS,
|
||||||
|
HConstants.DEFAULT_HBASE_META_VERSIONS))
|
||||||
|
.setInMemory(true)
|
||||||
|
.setBlocksize(conf.getInt(HConstants.HBASE_META_BLOCK_SIZE,
|
||||||
|
HConstants.DEFAULT_HBASE_META_BLOCK_SIZE))
|
||||||
|
.setScope(HConstants.REPLICATION_SCOPE_LOCAL)
|
||||||
|
// Disable blooms for meta. Needs work. Seems to mess w/ getClosestOrBefore.
|
||||||
|
.setBloomFilterType(BloomType.NONE)
|
||||||
|
.build())
|
||||||
.setCoprocessor(CoprocessorDescriptorBuilder.newBuilder(
|
.setCoprocessor(CoprocessorDescriptorBuilder.newBuilder(
|
||||||
MultiRowMutationEndpoint.class.getName())
|
MultiRowMutationEndpoint.class.getName())
|
||||||
.setPriority(Coprocessor.PRIORITY_SYSTEM).build());
|
.setPriority(Coprocessor.PRIORITY_SYSTEM).build());
|
||||||
|
|
|
@ -3436,7 +3436,6 @@ public class HBaseTestingUtility extends HBaseZKTestingUtility {
|
||||||
*/
|
*/
|
||||||
public void waitUntilAllSystemRegionsAssigned() throws IOException {
|
public void waitUntilAllSystemRegionsAssigned() throws IOException {
|
||||||
waitUntilAllRegionsAssigned(TableName.META_TABLE_NAME);
|
waitUntilAllRegionsAssigned(TableName.META_TABLE_NAME);
|
||||||
waitUntilAllRegionsAssigned(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
|
|
@ -58,8 +58,8 @@ public class TestGlobalMemStoreSize {
|
||||||
private static final Logger LOG = LoggerFactory.getLogger(TestGlobalMemStoreSize.class);
|
private static final Logger LOG = LoggerFactory.getLogger(TestGlobalMemStoreSize.class);
|
||||||
private static int regionServerNum = 4;
|
private static int regionServerNum = 4;
|
||||||
private static int regionNum = 16;
|
private static int regionNum = 16;
|
||||||
// total region num = region num + root and meta regions
|
// total region num = region num + meta regions
|
||||||
private static int totalRegionNum = regionNum+2;
|
private static int totalRegionNum = regionNum + 1;
|
||||||
|
|
||||||
private HBaseTestingUtility TEST_UTIL;
|
private HBaseTestingUtility TEST_UTIL;
|
||||||
private MiniHBaseCluster cluster;
|
private MiniHBaseCluster cluster;
|
||||||
|
|
|
@ -19,7 +19,6 @@ package org.apache.hadoop.hbase;
|
||||||
|
|
||||||
import static org.junit.Assert.assertEquals;
|
import static org.junit.Assert.assertEquals;
|
||||||
import static org.junit.Assert.assertNotNull;
|
import static org.junit.Assert.assertNotNull;
|
||||||
import static org.junit.Assert.assertNull;
|
|
||||||
import static org.junit.Assert.assertTrue;
|
import static org.junit.Assert.assertTrue;
|
||||||
import static org.junit.Assert.fail;
|
import static org.junit.Assert.fail;
|
||||||
|
|
||||||
|
@ -63,7 +62,6 @@ public class TestNamespace {
|
||||||
private static HBaseTestingUtility TEST_UTIL;
|
private static HBaseTestingUtility TEST_UTIL;
|
||||||
protected static Admin admin;
|
protected static Admin admin;
|
||||||
protected static HBaseCluster cluster;
|
protected static HBaseCluster cluster;
|
||||||
private static ZKNamespaceManager zkNamespaceManager;
|
|
||||||
private String prefix = "TestNamespace";
|
private String prefix = "TestNamespace";
|
||||||
|
|
||||||
@Rule
|
@Rule
|
||||||
|
@ -76,9 +74,6 @@ public class TestNamespace {
|
||||||
admin = TEST_UTIL.getAdmin();
|
admin = TEST_UTIL.getAdmin();
|
||||||
cluster = TEST_UTIL.getHBaseCluster();
|
cluster = TEST_UTIL.getHBaseCluster();
|
||||||
master = ((MiniHBaseCluster)cluster).getMaster();
|
master = ((MiniHBaseCluster)cluster).getMaster();
|
||||||
zkNamespaceManager =
|
|
||||||
new ZKNamespaceManager(master.getZooKeeper());
|
|
||||||
zkNamespaceManager.start();
|
|
||||||
LOG.info("Done initializing cluster");
|
LOG.info("Done initializing cluster");
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -107,19 +102,16 @@ public class TestNamespace {
|
||||||
admin.getNamespaceDescriptor(NamespaceDescriptor.DEFAULT_NAMESPACE.getName());
|
admin.getNamespaceDescriptor(NamespaceDescriptor.DEFAULT_NAMESPACE.getName());
|
||||||
assertNotNull(ns);
|
assertNotNull(ns);
|
||||||
assertEquals(ns.getName(), NamespaceDescriptor.DEFAULT_NAMESPACE.getName());
|
assertEquals(ns.getName(), NamespaceDescriptor.DEFAULT_NAMESPACE.getName());
|
||||||
assertNotNull(zkNamespaceManager.get(NamespaceDescriptor.DEFAULT_NAMESPACE_NAME_STR));
|
|
||||||
|
|
||||||
ns = admin.getNamespaceDescriptor(NamespaceDescriptor.SYSTEM_NAMESPACE.getName());
|
ns = admin.getNamespaceDescriptor(NamespaceDescriptor.SYSTEM_NAMESPACE.getName());
|
||||||
assertNotNull(ns);
|
assertNotNull(ns);
|
||||||
assertEquals(ns.getName(), NamespaceDescriptor.SYSTEM_NAMESPACE.getName());
|
assertEquals(ns.getName(), NamespaceDescriptor.SYSTEM_NAMESPACE.getName());
|
||||||
assertNotNull(zkNamespaceManager.get(NamespaceDescriptor.SYSTEM_NAMESPACE_NAME_STR));
|
|
||||||
|
|
||||||
assertEquals(2, admin.listNamespaceDescriptors().length);
|
assertEquals(2, admin.listNamespaceDescriptors().length);
|
||||||
|
|
||||||
//verify existence of system tables
|
//verify existence of system tables
|
||||||
Set<TableName> systemTables = Sets.newHashSet(
|
Set<TableName> systemTables = Sets.newHashSet(
|
||||||
TableName.META_TABLE_NAME,
|
TableName.META_TABLE_NAME);
|
||||||
TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
HTableDescriptor[] descs =
|
HTableDescriptor[] descs =
|
||||||
admin.listTableDescriptorsByNamespace(NamespaceDescriptor.SYSTEM_NAMESPACE.getName());
|
admin.listTableDescriptorsByNamespace(NamespaceDescriptor.SYSTEM_NAMESPACE.getName());
|
||||||
assertEquals(systemTables.size(), descs.length);
|
assertEquals(systemTables.size(), descs.length);
|
||||||
|
@ -181,18 +173,9 @@ public class TestNamespace {
|
||||||
//create namespace and verify
|
//create namespace and verify
|
||||||
admin.createNamespace(NamespaceDescriptor.create(nsName).build());
|
admin.createNamespace(NamespaceDescriptor.create(nsName).build());
|
||||||
assertEquals(3, admin.listNamespaceDescriptors().length);
|
assertEquals(3, admin.listNamespaceDescriptors().length);
|
||||||
TEST_UTIL.waitFor(60000, new Waiter.Predicate<Exception>() {
|
|
||||||
@Override
|
|
||||||
public boolean evaluate() throws Exception {
|
|
||||||
return zkNamespaceManager.list().size() == 3;
|
|
||||||
}
|
|
||||||
});
|
|
||||||
assertNotNull(zkNamespaceManager.get(nsName));
|
|
||||||
//remove namespace and verify
|
//remove namespace and verify
|
||||||
admin.deleteNamespace(nsName);
|
admin.deleteNamespace(nsName);
|
||||||
assertEquals(2, admin.listNamespaceDescriptors().length);
|
assertEquals(2, admin.listNamespaceDescriptors().length);
|
||||||
assertEquals(2, zkNamespaceManager.list().size());
|
|
||||||
assertNull(zkNamespaceManager.get(nsName));
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
|
|
|
@ -19,8 +19,6 @@ package org.apache.hadoop.hbase.client;
|
||||||
|
|
||||||
import static org.apache.hadoop.hbase.client.AsyncProcess.START_LOG_ERRORS_AFTER_COUNT_KEY;
|
import static org.apache.hadoop.hbase.client.AsyncProcess.START_LOG_ERRORS_AFTER_COUNT_KEY;
|
||||||
import static org.junit.Assert.assertEquals;
|
import static org.junit.Assert.assertEquals;
|
||||||
import static org.junit.Assert.assertNotNull;
|
|
||||||
import static org.junit.Assert.assertNull;
|
|
||||||
import static org.junit.Assert.assertTrue;
|
import static org.junit.Assert.assertTrue;
|
||||||
import static org.junit.Assert.fail;
|
import static org.junit.Assert.fail;
|
||||||
|
|
||||||
|
@ -29,14 +27,10 @@ import org.apache.hadoop.hbase.HBaseClassTestRule;
|
||||||
import org.apache.hadoop.hbase.HColumnDescriptor;
|
import org.apache.hadoop.hbase.HColumnDescriptor;
|
||||||
import org.apache.hadoop.hbase.HConstants;
|
import org.apache.hadoop.hbase.HConstants;
|
||||||
import org.apache.hadoop.hbase.HTableDescriptor;
|
import org.apache.hadoop.hbase.HTableDescriptor;
|
||||||
import org.apache.hadoop.hbase.MiniHBaseCluster;
|
|
||||||
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
import org.apache.hadoop.hbase.NamespaceExistException;
|
import org.apache.hadoop.hbase.NamespaceExistException;
|
||||||
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
import org.apache.hadoop.hbase.NamespaceNotFoundException;
|
||||||
import org.apache.hadoop.hbase.TableName;
|
import org.apache.hadoop.hbase.TableName;
|
||||||
import org.apache.hadoop.hbase.Waiter;
|
|
||||||
import org.apache.hadoop.hbase.ZKNamespaceManager;
|
|
||||||
import org.apache.hadoop.hbase.master.HMaster;
|
|
||||||
import org.apache.hadoop.hbase.testclassification.ClientTests;
|
import org.apache.hadoop.hbase.testclassification.ClientTests;
|
||||||
import org.apache.hadoop.hbase.testclassification.LargeTests;
|
import org.apache.hadoop.hbase.testclassification.LargeTests;
|
||||||
import org.junit.BeforeClass;
|
import org.junit.BeforeClass;
|
||||||
|
@ -58,8 +52,6 @@ public class TestAsyncNamespaceAdminApi extends TestAsyncAdminBase {
|
||||||
HBaseClassTestRule.forClass(TestAsyncNamespaceAdminApi.class);
|
HBaseClassTestRule.forClass(TestAsyncNamespaceAdminApi.class);
|
||||||
|
|
||||||
private String prefix = "TestNamespace";
|
private String prefix = "TestNamespace";
|
||||||
private static HMaster master;
|
|
||||||
private static ZKNamespaceManager zkNamespaceManager;
|
|
||||||
|
|
||||||
@BeforeClass
|
@BeforeClass
|
||||||
public static void setUpBeforeClass() throws Exception {
|
public static void setUpBeforeClass() throws Exception {
|
||||||
|
@ -69,9 +61,6 @@ public class TestAsyncNamespaceAdminApi extends TestAsyncAdminBase {
|
||||||
TEST_UTIL.getConfiguration().setInt(START_LOG_ERRORS_AFTER_COUNT_KEY, 0);
|
TEST_UTIL.getConfiguration().setInt(START_LOG_ERRORS_AFTER_COUNT_KEY, 0);
|
||||||
TEST_UTIL.startMiniCluster(1);
|
TEST_UTIL.startMiniCluster(1);
|
||||||
ASYNC_CONN = ConnectionFactory.createAsyncConnection(TEST_UTIL.getConfiguration()).get();
|
ASYNC_CONN = ConnectionFactory.createAsyncConnection(TEST_UTIL.getConfiguration()).get();
|
||||||
master = TEST_UTIL.getHBaseCluster().getMaster();
|
|
||||||
zkNamespaceManager = new ZKNamespaceManager(master.getZooKeeper());
|
|
||||||
zkNamespaceManager.start();
|
|
||||||
LOG.info("Done initializing cluster");
|
LOG.info("Done initializing cluster");
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -83,18 +72,9 @@ public class TestAsyncNamespaceAdminApi extends TestAsyncAdminBase {
|
||||||
// create namespace and verify
|
// create namespace and verify
|
||||||
admin.createNamespace(NamespaceDescriptor.create(nsName).build()).join();
|
admin.createNamespace(NamespaceDescriptor.create(nsName).build()).join();
|
||||||
assertEquals(3, admin.listNamespaceDescriptors().get().size());
|
assertEquals(3, admin.listNamespaceDescriptors().get().size());
|
||||||
TEST_UTIL.waitFor(60000, new Waiter.Predicate<Exception>() {
|
|
||||||
@Override
|
|
||||||
public boolean evaluate() throws Exception {
|
|
||||||
return zkNamespaceManager.list().size() == 3;
|
|
||||||
}
|
|
||||||
});
|
|
||||||
assertNotNull(zkNamespaceManager.get(nsName));
|
|
||||||
// delete namespace and verify
|
// delete namespace and verify
|
||||||
admin.deleteNamespace(nsName).join();
|
admin.deleteNamespace(nsName).join();
|
||||||
assertEquals(2, admin.listNamespaceDescriptors().get().size());
|
assertEquals(2, admin.listNamespaceDescriptors().get().size());
|
||||||
assertEquals(2, zkNamespaceManager.list().size());
|
|
||||||
assertNull(zkNamespaceManager.get(nsName));
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
|
|
|
@ -6396,7 +6396,7 @@ public class TestFromClientSide {
|
||||||
scan.setCaching(1);
|
scan.setCaching(1);
|
||||||
// Filter out any records
|
// Filter out any records
|
||||||
scan.setFilter(new FilterList(new FirstKeyOnlyFilter(), new InclusiveStopFilter(new byte[0])));
|
scan.setFilter(new FilterList(new FirstKeyOnlyFilter(), new InclusiveStopFilter(new byte[0])));
|
||||||
try (Table table = TEST_UTIL.getConnection().getTable(TableName.NAMESPACE_TABLE_NAME)) {
|
try (Table table = TEST_UTIL.getConnection().getTable(TableName.META_TABLE_NAME)) {
|
||||||
try (ResultScanner s = table.getScanner(scan)) {
|
try (ResultScanner s = table.getScanner(scan)) {
|
||||||
assertNull(s.next());
|
assertNull(s.next());
|
||||||
}
|
}
|
||||||
|
|
|
@ -505,19 +505,19 @@ public abstract class AbstractTestDLS {
|
||||||
LOG.debug("Waiting for no more RIT\n");
|
LOG.debug("Waiting for no more RIT\n");
|
||||||
blockUntilNoRIT(zkw, master);
|
blockUntilNoRIT(zkw, master);
|
||||||
NavigableSet<String> regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
NavigableSet<String> regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
||||||
LOG.debug("Verifying only catalog and namespace regions are assigned\n");
|
LOG.debug("Verifying only catalog region is assigned\n");
|
||||||
if (regions.size() != 2) {
|
if (regions.size() != 1) {
|
||||||
for (String oregion : regions)
|
for (String oregion : regions)
|
||||||
LOG.debug("Region still online: " + oregion);
|
LOG.debug("Region still online: " + oregion);
|
||||||
}
|
}
|
||||||
assertEquals(2 + existingRegions, regions.size());
|
assertEquals(1 + existingRegions, regions.size());
|
||||||
LOG.debug("Enabling table\n");
|
LOG.debug("Enabling table\n");
|
||||||
TEST_UTIL.getAdmin().enableTable(tableName);
|
TEST_UTIL.getAdmin().enableTable(tableName);
|
||||||
LOG.debug("Waiting for no more RIT\n");
|
LOG.debug("Waiting for no more RIT\n");
|
||||||
blockUntilNoRIT(zkw, master);
|
blockUntilNoRIT(zkw, master);
|
||||||
LOG.debug("Verifying there are " + numRegions + " assigned on cluster\n");
|
LOG.debug("Verifying there are " + numRegions + " assigned on cluster\n");
|
||||||
regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
||||||
assertEquals(numRegions + 2 + existingRegions, regions.size());
|
assertEquals(numRegions + 1 + existingRegions, regions.size());
|
||||||
return table;
|
return table;
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
|
@ -20,7 +20,6 @@ package org.apache.hadoop.hbase.master;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import org.apache.hadoop.conf.Configuration;
|
import org.apache.hadoop.conf.Configuration;
|
||||||
import org.apache.hadoop.hbase.CompatibilityFactory;
|
import org.apache.hadoop.hbase.CompatibilityFactory;
|
||||||
import org.apache.hadoop.hbase.CoordinatedStateManager;
|
|
||||||
import org.apache.hadoop.hbase.HBaseClassTestRule;
|
import org.apache.hadoop.hbase.HBaseClassTestRule;
|
||||||
import org.apache.hadoop.hbase.HBaseTestingUtility;
|
import org.apache.hadoop.hbase.HBaseTestingUtility;
|
||||||
import org.apache.hadoop.hbase.MiniHBaseCluster;
|
import org.apache.hadoop.hbase.MiniHBaseCluster;
|
||||||
|
@ -42,40 +41,51 @@ import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos;
|
||||||
import org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos;
|
import org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos;
|
||||||
|
|
||||||
@Category({MasterTests.class, MediumTests.class})
|
@Category({ MasterTests.class, MediumTests.class })
|
||||||
public class TestMasterMetrics {
|
public class TestMasterMetrics {
|
||||||
|
|
||||||
@ClassRule
|
@ClassRule
|
||||||
public static final HBaseClassTestRule CLASS_RULE =
|
public static final HBaseClassTestRule CLASS_RULE =
|
||||||
HBaseClassTestRule.forClass(TestMasterMetrics.class);
|
HBaseClassTestRule.forClass(TestMasterMetrics.class);
|
||||||
|
|
||||||
private static final Logger LOG = LoggerFactory.getLogger(TestMasterMetrics.class);
|
private static final Logger LOG = LoggerFactory.getLogger(TestMasterMetrics.class);
|
||||||
private static final MetricsAssertHelper metricsHelper = CompatibilityFactory
|
private static final MetricsAssertHelper metricsHelper =
|
||||||
.getInstance(MetricsAssertHelper.class);
|
CompatibilityFactory.getInstance(MetricsAssertHelper.class);
|
||||||
|
|
||||||
private static MiniHBaseCluster cluster;
|
private static MiniHBaseCluster cluster;
|
||||||
private static HMaster master;
|
private static HMaster master;
|
||||||
private static HBaseTestingUtility TEST_UTIL;
|
private static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
|
||||||
|
|
||||||
public static class MyMaster extends HMaster {
|
public static class MyMaster extends HMaster {
|
||||||
|
|
||||||
public MyMaster(Configuration conf) throws IOException, KeeperException, InterruptedException {
|
public MyMaster(Configuration conf) throws IOException, KeeperException, InterruptedException {
|
||||||
super(conf);
|
super(conf);
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
protected void tryRegionServerReport(
|
protected void tryRegionServerReport(long reportStartTime, long reportEndTime) {
|
||||||
long reportStartTime, long reportEndTime) {
|
|
||||||
// do nothing
|
// do nothing
|
||||||
}
|
}
|
||||||
|
}
|
||||||
|
|
||||||
|
public static class MyRegionServer extends MiniHBaseCluster.MiniHBaseClusterRegionServer {
|
||||||
|
|
||||||
|
public MyRegionServer(Configuration conf) throws IOException, InterruptedException {
|
||||||
|
super(conf);
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override
|
||||||
|
protected void tryRegionServerReport(long reportStartTime, long reportEndTime) {
|
||||||
|
// do nothing
|
||||||
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
@BeforeClass
|
@BeforeClass
|
||||||
public static void startCluster() throws Exception {
|
public static void startCluster() throws Exception {
|
||||||
LOG.info("Starting cluster");
|
LOG.info("Starting cluster");
|
||||||
TEST_UTIL = new HBaseTestingUtility();
|
|
||||||
// Set master class and use default values for other options.
|
// Set master class and use default values for other options.
|
||||||
StartMiniClusterOption option = StartMiniClusterOption.builder()
|
StartMiniClusterOption option = StartMiniClusterOption.builder().masterClass(MyMaster.class)
|
||||||
.masterClass(MyMaster.class).build();
|
.rsClass(MyRegionServer.class).build();
|
||||||
TEST_UTIL.startMiniCluster(option);
|
TEST_UTIL.startMiniCluster(option);
|
||||||
cluster = TEST_UTIL.getHBaseCluster();
|
cluster = TEST_UTIL.getHBaseCluster();
|
||||||
LOG.info("Waiting for active/ready master");
|
LOG.info("Waiting for active/ready master");
|
||||||
|
@ -85,61 +95,44 @@ public class TestMasterMetrics {
|
||||||
|
|
||||||
@AfterClass
|
@AfterClass
|
||||||
public static void after() throws Exception {
|
public static void after() throws Exception {
|
||||||
if (TEST_UTIL != null) {
|
master.stopMaster();
|
||||||
TEST_UTIL.shutdownMiniCluster();
|
TEST_UTIL.shutdownMiniCluster();
|
||||||
}
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
public void testClusterRequests() throws Exception {
|
public void testClusterRequests() throws Exception {
|
||||||
|
|
||||||
// sending fake request to master to see how metric value has changed
|
// sending fake request to master to see how metric value has changed
|
||||||
|
|
||||||
RegionServerStatusProtos.RegionServerReportRequest.Builder request =
|
RegionServerStatusProtos.RegionServerReportRequest.Builder request =
|
||||||
RegionServerStatusProtos.RegionServerReportRequest.newBuilder();
|
RegionServerStatusProtos.RegionServerReportRequest.newBuilder();
|
||||||
ServerName serverName = cluster.getMaster(0).getServerName();
|
ServerName serverName = cluster.getMaster(0).getServerName();
|
||||||
request.setServer(ProtobufUtil.toServerName(serverName));
|
request.setServer(ProtobufUtil.toServerName(serverName));
|
||||||
long expectedRequestNumber = 10000;
|
long expectedRequestNumber = 10000;
|
||||||
|
|
||||||
MetricsMasterSource masterSource = master.getMasterMetrics().getMetricsSource();
|
MetricsMasterSource masterSource = master.getMasterMetrics().getMetricsSource();
|
||||||
ClusterStatusProtos.ServerLoad sl = ClusterStatusProtos.ServerLoad.newBuilder()
|
ClusterStatusProtos.ServerLoad sl = ClusterStatusProtos.ServerLoad.newBuilder()
|
||||||
.setTotalNumberOfRequests(expectedRequestNumber)
|
.setTotalNumberOfRequests(expectedRequestNumber).build();
|
||||||
.build();
|
|
||||||
request.setLoad(sl);
|
request.setLoad(sl);
|
||||||
|
|
||||||
master.getMasterRpcServices().regionServerReport(null, request.build());
|
master.getMasterRpcServices().regionServerReport(null, request.build());
|
||||||
boolean tablesOnMaster = LoadBalancer.isTablesOnMaster(TEST_UTIL.getConfiguration());
|
metricsHelper.assertCounter("cluster_requests", expectedRequestNumber, masterSource);
|
||||||
if (tablesOnMaster) {
|
|
||||||
metricsHelper.assertCounter("cluster_requests", expectedRequestNumber, masterSource);
|
|
||||||
} else {
|
|
||||||
metricsHelper.assertCounterGt("cluster_requests", expectedRequestNumber, masterSource);
|
|
||||||
|
|
||||||
}
|
|
||||||
|
|
||||||
expectedRequestNumber = 15000;
|
expectedRequestNumber = 15000;
|
||||||
|
|
||||||
sl = ClusterStatusProtos.ServerLoad.newBuilder()
|
sl = ClusterStatusProtos.ServerLoad.newBuilder().setTotalNumberOfRequests(expectedRequestNumber)
|
||||||
.setTotalNumberOfRequests(expectedRequestNumber)
|
.build();
|
||||||
.build();
|
|
||||||
request.setLoad(sl);
|
request.setLoad(sl);
|
||||||
|
|
||||||
master.getMasterRpcServices().regionServerReport(null, request.build());
|
master.getMasterRpcServices().regionServerReport(null, request.build());
|
||||||
if (tablesOnMaster) {
|
metricsHelper.assertCounter("cluster_requests", expectedRequestNumber, masterSource);
|
||||||
metricsHelper.assertCounter("cluster_requests", expectedRequestNumber, masterSource);
|
|
||||||
} else {
|
|
||||||
metricsHelper.assertCounterGt("cluster_requests", expectedRequestNumber, masterSource);
|
|
||||||
}
|
|
||||||
|
|
||||||
master.stopMaster();
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
public void testDefaultMasterMetrics() throws Exception {
|
public void testDefaultMasterMetrics() throws Exception {
|
||||||
MetricsMasterSource masterSource = master.getMasterMetrics().getMetricsSource();
|
MetricsMasterSource masterSource = master.getMasterMetrics().getMetricsSource();
|
||||||
boolean tablesOnMaster = LoadBalancer.isTablesOnMaster(TEST_UTIL.getConfiguration());
|
boolean tablesOnMaster = LoadBalancer.isTablesOnMaster(TEST_UTIL.getConfiguration());
|
||||||
metricsHelper.assertGauge( "numRegionServers",1 + (tablesOnMaster? 1: 0), masterSource);
|
metricsHelper.assertGauge("numRegionServers", 1 + (tablesOnMaster ? 1 : 0), masterSource);
|
||||||
metricsHelper.assertGauge( "averageLoad", 1 + (tablesOnMaster? 0: 1), masterSource);
|
metricsHelper.assertGauge("averageLoad", 1, masterSource);
|
||||||
metricsHelper.assertGauge( "numDeadRegionServers", 0, masterSource);
|
metricsHelper.assertGauge("numDeadRegionServers", 0, masterSource);
|
||||||
|
|
||||||
metricsHelper.assertGauge("masterStartTime", master.getMasterStartTime(), masterSource);
|
metricsHelper.assertGauge("masterStartTime", master.getMasterStartTime(), masterSource);
|
||||||
metricsHelper.assertGauge("masterActiveTime", master.getMasterActiveTime(), masterSource);
|
metricsHelper.assertGauge("masterActiveTime", master.getMasterActiveTime(), masterSource);
|
||||||
|
|
|
@ -200,7 +200,7 @@ public class TestMasterOperationsForRegionReplicas {
|
||||||
StartMiniClusterOption option = StartMiniClusterOption.builder()
|
StartMiniClusterOption option = StartMiniClusterOption.builder()
|
||||||
.numRegionServers(numSlaves).rsPorts(rsports).build();
|
.numRegionServers(numSlaves).rsPorts(rsports).build();
|
||||||
TEST_UTIL.startMiniHBaseCluster(option);
|
TEST_UTIL.startMiniHBaseCluster(option);
|
||||||
TEST_UTIL.waitTableEnabled(tableName);
|
TEST_UTIL.waitTableAvailable(tableName);
|
||||||
validateFromSnapshotFromMeta(TEST_UTIL, tableName, numRegions, numReplica,
|
validateFromSnapshotFromMeta(TEST_UTIL, tableName, numRegions, numReplica,
|
||||||
ADMIN.getConnection());
|
ADMIN.getConnection());
|
||||||
|
|
||||||
|
@ -208,7 +208,7 @@ public class TestMasterOperationsForRegionReplicas {
|
||||||
// one server running
|
// one server running
|
||||||
TEST_UTIL.shutdownMiniHBaseCluster();
|
TEST_UTIL.shutdownMiniHBaseCluster();
|
||||||
TEST_UTIL.startMiniHBaseCluster();
|
TEST_UTIL.startMiniHBaseCluster();
|
||||||
TEST_UTIL.waitTableEnabled(tableName);
|
TEST_UTIL.waitTableAvailable(tableName);
|
||||||
validateSingleRegionServerAssignment(ADMIN.getConnection(), numRegions, numReplica);
|
validateSingleRegionServerAssignment(ADMIN.getConnection(), numRegions, numReplica);
|
||||||
for (int i = 1; i < numSlaves; i++) { //restore the cluster
|
for (int i = 1; i < numSlaves; i++) { //restore the cluster
|
||||||
TEST_UTIL.getMiniHBaseCluster().startRegionServer();
|
TEST_UTIL.getMiniHBaseCluster().startRegionServer();
|
||||||
|
@ -334,7 +334,7 @@ public class TestMasterOperationsForRegionReplicas {
|
||||||
connection);
|
connection);
|
||||||
snapshot.initialize();
|
snapshot.initialize();
|
||||||
Map<RegionInfo, ServerName> regionToServerMap = snapshot.getRegionToRegionServerMap();
|
Map<RegionInfo, ServerName> regionToServerMap = snapshot.getRegionToRegionServerMap();
|
||||||
assert(regionToServerMap.size() == numRegions * numReplica + 1); //'1' for the namespace
|
assert(regionToServerMap.size() == numRegions * numReplica);
|
||||||
Map<ServerName, List<RegionInfo>> serverToRegionMap = snapshot.getRegionServerToRegionMap();
|
Map<ServerName, List<RegionInfo>> serverToRegionMap = snapshot.getRegionServerToRegionMap();
|
||||||
for (Map.Entry<ServerName, List<RegionInfo>> entry : serverToRegionMap.entrySet()) {
|
for (Map.Entry<ServerName, List<RegionInfo>> entry : serverToRegionMap.entrySet()) {
|
||||||
if (entry.getKey().equals(util.getHBaseCluster().getMaster().getServerName())) {
|
if (entry.getKey().equals(util.getHBaseCluster().getMaster().getServerName())) {
|
||||||
|
@ -361,14 +361,14 @@ public class TestMasterOperationsForRegionReplicas {
|
||||||
connection);
|
connection);
|
||||||
snapshot.initialize();
|
snapshot.initialize();
|
||||||
Map<RegionInfo, ServerName> regionToServerMap = snapshot.getRegionToRegionServerMap();
|
Map<RegionInfo, ServerName> regionToServerMap = snapshot.getRegionToRegionServerMap();
|
||||||
assertEquals(regionToServerMap.size(), numRegions * numReplica + 1);
|
assertEquals(regionToServerMap.size(), numRegions * numReplica);
|
||||||
Map<ServerName, List<RegionInfo>> serverToRegionMap = snapshot.getRegionServerToRegionMap();
|
Map<ServerName, List<RegionInfo>> serverToRegionMap = snapshot.getRegionServerToRegionMap();
|
||||||
assertEquals("One Region Only", 1, serverToRegionMap.keySet().size());
|
assertEquals("One Region Only", 1, serverToRegionMap.keySet().size());
|
||||||
for (Map.Entry<ServerName, List<RegionInfo>> entry : serverToRegionMap.entrySet()) {
|
for (Map.Entry<ServerName, List<RegionInfo>> entry : serverToRegionMap.entrySet()) {
|
||||||
if (entry.getKey().equals(TEST_UTIL.getHBaseCluster().getMaster().getServerName())) {
|
if (entry.getKey().equals(TEST_UTIL.getHBaseCluster().getMaster().getServerName())) {
|
||||||
continue;
|
continue;
|
||||||
}
|
}
|
||||||
assertEquals(entry.getValue().size(), numRegions * numReplica + 1);
|
assertEquals(entry.getValue().size(), numRegions * numReplica);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
|
@ -91,9 +91,8 @@ public class TestMasterRestartAfterDisablingTable {
|
||||||
TEST_UTIL.getAdmin().disableTable(tableName);
|
TEST_UTIL.getAdmin().disableTable(tableName);
|
||||||
|
|
||||||
NavigableSet<String> regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
NavigableSet<String> regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
||||||
assertEquals(
|
assertEquals("The number of regions for the table tableRestart should be 0 and only" +
|
||||||
"The number of regions for the table tableRestart should be 0 and only"
|
"the catalog table should be present.", 1, regions.size());
|
||||||
+ "the catalog and namespace tables should be present.", 2, regions.size());
|
|
||||||
|
|
||||||
List<MasterThread> masterThreads = cluster.getMasterThreads();
|
List<MasterThread> masterThreads = cluster.getMasterThreads();
|
||||||
MasterThread activeMaster = null;
|
MasterThread activeMaster = null;
|
||||||
|
@ -120,12 +119,10 @@ public class TestMasterRestartAfterDisablingTable {
|
||||||
TEST_UTIL.waitUntilNoRegionsInTransition(60000);
|
TEST_UTIL.waitUntilNoRegionsInTransition(60000);
|
||||||
log("Verifying there are " + numRegions + " assigned on cluster\n");
|
log("Verifying there are " + numRegions + " assigned on cluster\n");
|
||||||
regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
||||||
assertEquals("The assigned regions were not onlined after master"
|
assertEquals("The assigned regions were not onlined after master" +
|
||||||
+ " switch except for the catalog and namespace tables.",
|
" switch except for the catalog table.", 5, regions.size());
|
||||||
6, regions.size());
|
assertTrue("The table should be in enabled state", cluster.getMaster().getTableStateManager()
|
||||||
assertTrue("The table should be in enabled state",
|
.isTableState(TableName.valueOf(name.getMethodName()), TableState.State.ENABLED));
|
||||||
cluster.getMaster().getTableStateManager()
|
|
||||||
.isTableState(TableName.valueOf(name.getMethodName()), TableState.State.ENABLED));
|
|
||||||
ht.close();
|
ht.close();
|
||||||
TEST_UTIL.shutdownMiniCluster();
|
TEST_UTIL.shutdownMiniCluster();
|
||||||
}
|
}
|
||||||
|
|
|
@ -0,0 +1,108 @@
|
||||||
|
/**
|
||||||
|
* Licensed to the Apache Software Foundation (ASF) under one
|
||||||
|
* or more contributor license agreements. See the NOTICE file
|
||||||
|
* distributed with this work for additional information
|
||||||
|
* regarding copyright ownership. The ASF licenses this file
|
||||||
|
* to you under the Apache License, Version 2.0 (the
|
||||||
|
* "License"); you may not use this file except in compliance
|
||||||
|
* with the License. You may obtain a copy of the License at
|
||||||
|
*
|
||||||
|
* http://www.apache.org/licenses/LICENSE-2.0
|
||||||
|
*
|
||||||
|
* Unless required by applicable law or agreed to in writing, software
|
||||||
|
* distributed under the License is distributed on an "AS IS" BASIS,
|
||||||
|
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
|
||||||
|
* See the License for the specific language governing permissions and
|
||||||
|
* limitations under the License.
|
||||||
|
*/
|
||||||
|
package org.apache.hadoop.hbase.master;
|
||||||
|
|
||||||
|
import static org.junit.Assert.assertEquals;
|
||||||
|
import static org.junit.Assert.assertFalse;
|
||||||
|
|
||||||
|
import java.io.IOException;
|
||||||
|
import org.apache.hadoop.hbase.HBaseClassTestRule;
|
||||||
|
import org.apache.hadoop.hbase.HBaseTestingUtility;
|
||||||
|
import org.apache.hadoop.hbase.NamespaceDescriptor;
|
||||||
|
import org.apache.hadoop.hbase.TableName;
|
||||||
|
import org.apache.hadoop.hbase.client.Put;
|
||||||
|
import org.apache.hadoop.hbase.client.Table;
|
||||||
|
import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
|
||||||
|
import org.apache.hadoop.hbase.master.procedure.AbstractStateMachineNamespaceProcedure;
|
||||||
|
import org.apache.hadoop.hbase.testclassification.LargeTests;
|
||||||
|
import org.apache.hadoop.hbase.testclassification.MasterTests;
|
||||||
|
import org.apache.hadoop.hbase.util.Bytes;
|
||||||
|
import org.apache.hadoop.hbase.util.JVMClusterUtil.MasterThread;
|
||||||
|
import org.junit.AfterClass;
|
||||||
|
import org.junit.BeforeClass;
|
||||||
|
import org.junit.ClassRule;
|
||||||
|
import org.junit.Test;
|
||||||
|
import org.junit.experimental.categories.Category;
|
||||||
|
|
||||||
|
import org.apache.hadoop.hbase.shaded.protobuf.ProtobufUtil;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Testcase for HBASE-21154.
|
||||||
|
*/
|
||||||
|
@Category({ MasterTests.class, LargeTests.class })
|
||||||
|
public class TestMigrateNamespaceTable {
|
||||||
|
|
||||||
|
@ClassRule
|
||||||
|
public static final HBaseClassTestRule CLASS_RULE =
|
||||||
|
HBaseClassTestRule.forClass(TestMigrateNamespaceTable.class);
|
||||||
|
|
||||||
|
private static final HBaseTestingUtility UTIL = new HBaseTestingUtility();
|
||||||
|
|
||||||
|
@BeforeClass
|
||||||
|
public static void setUp() throws Exception {
|
||||||
|
UTIL.startMiniCluster(1);
|
||||||
|
}
|
||||||
|
|
||||||
|
@AfterClass
|
||||||
|
public static void tearDow() throws Exception {
|
||||||
|
UTIL.shutdownMiniCluster();
|
||||||
|
}
|
||||||
|
|
||||||
|
@Test
|
||||||
|
public void testMigrate() throws IOException, InterruptedException {
|
||||||
|
UTIL.getAdmin().createTable(TableDescriptorBuilder.NAMESPACE_TABLEDESC);
|
||||||
|
try (Table table = UTIL.getConnection().getTable(TableName.NAMESPACE_TABLE_NAME)) {
|
||||||
|
for (int i = 0; i < 5; i++) {
|
||||||
|
NamespaceDescriptor nd = NamespaceDescriptor.create("Test-NS-" + i)
|
||||||
|
.addConfiguration("key-" + i, "value-" + i).build();
|
||||||
|
table.put(new Put(Bytes.toBytes(nd.getName())).addColumn(
|
||||||
|
TableDescriptorBuilder.NAMESPACE_FAMILY_INFO_BYTES,
|
||||||
|
TableDescriptorBuilder.NAMESPACE_COL_DESC_BYTES,
|
||||||
|
ProtobufUtil.toProtoNamespaceDescriptor(nd).toByteArray()));
|
||||||
|
AbstractStateMachineNamespaceProcedure
|
||||||
|
.createDirectory(UTIL.getMiniHBaseCluster().getMaster().getMasterFileSystem(), nd);
|
||||||
|
}
|
||||||
|
}
|
||||||
|
MasterThread masterThread = UTIL.getMiniHBaseCluster().getMasterThread();
|
||||||
|
masterThread.getMaster().stop("For testing");
|
||||||
|
masterThread.join();
|
||||||
|
UTIL.getMiniHBaseCluster().startMaster();
|
||||||
|
// 5 + default and system('hbase')
|
||||||
|
assertEquals(7, UTIL.getAdmin().listNamespaceDescriptors().length);
|
||||||
|
for (int i = 0; i < 5; i++) {
|
||||||
|
NamespaceDescriptor nd = UTIL.getAdmin().getNamespaceDescriptor("Test-NS-" + i);
|
||||||
|
assertEquals("Test-NS-" + i, nd.getName());
|
||||||
|
assertEquals(1, nd.getConfiguration().size());
|
||||||
|
assertEquals("value-" + i, nd.getConfigurationValue("key-" + i));
|
||||||
|
}
|
||||||
|
UTIL.waitFor(30000, () -> UTIL.getAdmin().isTableDisabled(TableName.NAMESPACE_TABLE_NAME));
|
||||||
|
|
||||||
|
masterThread = UTIL.getMiniHBaseCluster().getMasterThread();
|
||||||
|
masterThread.getMaster().stop("For testing");
|
||||||
|
masterThread.join();
|
||||||
|
|
||||||
|
UTIL.getMiniHBaseCluster().startMaster();
|
||||||
|
|
||||||
|
// make sure that we could still restart the cluster after disabling the namespace table.
|
||||||
|
assertEquals(7, UTIL.getAdmin().listNamespaceDescriptors().length);
|
||||||
|
|
||||||
|
// let's delete the namespace table
|
||||||
|
UTIL.getAdmin().deleteTable(TableName.NAMESPACE_TABLE_NAME);
|
||||||
|
assertFalse(UTIL.getAdmin().tableExists(TableName.NAMESPACE_TABLE_NAME));
|
||||||
|
}
|
||||||
|
}
|
|
@ -81,7 +81,7 @@ public class TestRestartCluster {
|
||||||
}
|
}
|
||||||
|
|
||||||
List<RegionInfo> allRegions = MetaTableAccessor.getAllRegions(UTIL.getConnection(), false);
|
List<RegionInfo> allRegions = MetaTableAccessor.getAllRegions(UTIL.getConnection(), false);
|
||||||
assertEquals(4, allRegions.size());
|
assertEquals(3, allRegions.size());
|
||||||
|
|
||||||
LOG.info("\n\nShutting down cluster");
|
LOG.info("\n\nShutting down cluster");
|
||||||
UTIL.shutdownMiniHBaseCluster();
|
UTIL.shutdownMiniHBaseCluster();
|
||||||
|
@ -96,7 +96,7 @@ public class TestRestartCluster {
|
||||||
// Otherwise we're reusing an Connection that has gone stale because
|
// Otherwise we're reusing an Connection that has gone stale because
|
||||||
// the shutdown of the cluster also called shut of the connection.
|
// the shutdown of the cluster also called shut of the connection.
|
||||||
allRegions = MetaTableAccessor.getAllRegions(UTIL.getConnection(), false);
|
allRegions = MetaTableAccessor.getAllRegions(UTIL.getConnection(), false);
|
||||||
assertEquals(4, allRegions.size());
|
assertEquals(3, allRegions.size());
|
||||||
LOG.info("\n\nWaiting for tables to be available");
|
LOG.info("\n\nWaiting for tables to be available");
|
||||||
for(TableName TABLE: TABLES) {
|
for(TableName TABLE: TABLES) {
|
||||||
try {
|
try {
|
||||||
|
@ -201,9 +201,6 @@ public class TestRestartCluster {
|
||||||
snapshot.getRegionToRegionServerMap();
|
snapshot.getRegionToRegionServerMap();
|
||||||
assertEquals(regionToRegionServerMap.size(), newRegionToRegionServerMap.size());
|
assertEquals(regionToRegionServerMap.size(), newRegionToRegionServerMap.size());
|
||||||
for (Map.Entry<RegionInfo, ServerName> entry : newRegionToRegionServerMap.entrySet()) {
|
for (Map.Entry<RegionInfo, ServerName> entry : newRegionToRegionServerMap.entrySet()) {
|
||||||
if (TableName.NAMESPACE_TABLE_NAME.equals(entry.getKey().getTable())) {
|
|
||||||
continue;
|
|
||||||
}
|
|
||||||
ServerName oldServer = regionToRegionServerMap.get(entry.getKey());
|
ServerName oldServer = regionToRegionServerMap.get(entry.getKey());
|
||||||
ServerName currentServer = entry.getValue();
|
ServerName currentServer = entry.getValue();
|
||||||
LOG.info(
|
LOG.info(
|
||||||
|
|
|
@ -103,11 +103,13 @@ public class TestRollingRestart {
|
||||||
log("Waiting for no more RIT\n");
|
log("Waiting for no more RIT\n");
|
||||||
TEST_UTIL.waitUntilNoRegionsInTransition(60000);
|
TEST_UTIL.waitUntilNoRegionsInTransition(60000);
|
||||||
NavigableSet<String> regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
NavigableSet<String> regions = HBaseTestingUtility.getAllOnlineRegions(cluster);
|
||||||
log("Verifying only catalog and namespace regions are assigned\n");
|
log("Verifying only catalog region is assigned\n");
|
||||||
if (regions.size() != 2) {
|
if (regions.size() != 1) {
|
||||||
for (String oregion : regions) log("Region still online: " + oregion);
|
for (String oregion : regions) {
|
||||||
|
log("Region still online: " + oregion);
|
||||||
|
}
|
||||||
}
|
}
|
||||||
assertEquals(2, regions.size());
|
assertEquals(1, regions.size());
|
||||||
log("Enabling table\n");
|
log("Enabling table\n");
|
||||||
TEST_UTIL.getAdmin().enableTable(tableName);
|
TEST_UTIL.getAdmin().enableTable(tableName);
|
||||||
log("Waiting for no more RIT\n");
|
log("Waiting for no more RIT\n");
|
||||||
|
|
|
@ -64,6 +64,7 @@ public class TestRegionMoveAndAbandon {
|
||||||
private MiniZooKeeperCluster zkCluster;
|
private MiniZooKeeperCluster zkCluster;
|
||||||
private HRegionServer rs1;
|
private HRegionServer rs1;
|
||||||
private HRegionServer rs2;
|
private HRegionServer rs2;
|
||||||
|
private TableName tableName;
|
||||||
private RegionInfo regionInfo;
|
private RegionInfo regionInfo;
|
||||||
|
|
||||||
@Before
|
@Before
|
||||||
|
@ -75,10 +76,10 @@ public class TestRegionMoveAndAbandon {
|
||||||
rs1 = cluster.getRegionServer(0);
|
rs1 = cluster.getRegionServer(0);
|
||||||
rs2 = cluster.getRegionServer(1);
|
rs2 = cluster.getRegionServer(1);
|
||||||
assertEquals(2, cluster.getRegionServerThreads().size());
|
assertEquals(2, cluster.getRegionServerThreads().size());
|
||||||
// We'll use hbase:namespace for our testing
|
tableName = TableName.valueOf(name.getMethodName());
|
||||||
UTIL.waitTableAvailable(TableName.NAMESPACE_TABLE_NAME, 30_000);
|
UTIL.createTable(tableName, Bytes.toBytes("cf"));
|
||||||
regionInfo =
|
UTIL.waitTableAvailable(tableName, 30_000);
|
||||||
Iterables.getOnlyElement(cluster.getRegions(TableName.NAMESPACE_TABLE_NAME)).getRegionInfo();
|
regionInfo = Iterables.getOnlyElement(cluster.getRegions(tableName)).getRegionInfo();
|
||||||
}
|
}
|
||||||
|
|
||||||
@After
|
@After
|
||||||
|
@ -105,7 +106,7 @@ public class TestRegionMoveAndAbandon {
|
||||||
// Stop RS1
|
// Stop RS1
|
||||||
cluster.killRegionServer(rs1.getServerName());
|
cluster.killRegionServer(rs1.getServerName());
|
||||||
// Region should get moved to RS2
|
// Region should get moved to RS2
|
||||||
UTIL.waitTableAvailable(TableName.NAMESPACE_TABLE_NAME, 30_000);
|
UTIL.waitTableAvailable(tableName, 30_000);
|
||||||
// Restart the master
|
// Restart the master
|
||||||
LOG.info("Killing master {}", cluster.getMaster().getServerName());
|
LOG.info("Killing master {}", cluster.getMaster().getServerName());
|
||||||
cluster.killMaster(cluster.getMaster().getServerName());
|
cluster.killMaster(cluster.getMaster().getServerName());
|
||||||
|
@ -120,7 +121,7 @@ public class TestRegionMoveAndAbandon {
|
||||||
UTIL.waitFor(30_000, new Waiter.Predicate<Exception>() {
|
UTIL.waitFor(30_000, new Waiter.Predicate<Exception>() {
|
||||||
@Override
|
@Override
|
||||||
public boolean evaluate() throws Exception {
|
public boolean evaluate() throws Exception {
|
||||||
try (Table nsTable = UTIL.getConnection().getTable(TableName.NAMESPACE_TABLE_NAME)) {
|
try (Table nsTable = UTIL.getConnection().getTable(tableName)) {
|
||||||
// Doesn't matter what we're getting. We just want to make sure we can access the region
|
// Doesn't matter what we're getting. We just want to make sure we can access the region
|
||||||
nsTable.get(new Get(Bytes.toBytes("a")));
|
nsTable.get(new Get(Bytes.toBytes("a")));
|
||||||
return true;
|
return true;
|
||||||
|
|
|
@ -160,7 +160,6 @@ public class TestFavoredStochasticLoadBalancer extends BalancerTestBase {
|
||||||
LoadBalancer balancer = master.getLoadBalancer();
|
LoadBalancer balancer = master.getLoadBalancer();
|
||||||
List<RegionInfo> regions = admin.getRegions(tableName);
|
List<RegionInfo> regions = admin.getRegions(tableName);
|
||||||
regions.addAll(admin.getTableRegions(TableName.META_TABLE_NAME));
|
regions.addAll(admin.getTableRegions(TableName.META_TABLE_NAME));
|
||||||
regions.addAll(admin.getTableRegions(TableName.NAMESPACE_TABLE_NAME));
|
|
||||||
List<ServerName> servers = Lists.newArrayList(
|
List<ServerName> servers = Lists.newArrayList(
|
||||||
admin.getClusterMetrics(EnumSet.of(Option.LIVE_SERVERS)).getLiveServerMetrics().keySet());
|
admin.getClusterMetrics(EnumSet.of(Option.LIVE_SERVERS)).getLiveServerMetrics().keySet());
|
||||||
Map<ServerName, List<RegionInfo>> map = balancer.roundRobinAssignment(regions, servers);
|
Map<ServerName, List<RegionInfo>> map = balancer.roundRobinAssignment(regions, servers);
|
||||||
|
|
|
@ -902,7 +902,7 @@ public class TestMasterProcedureScheduler {
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public TableName getTableName() {
|
public TableName getTableName() {
|
||||||
return TableName.NAMESPACE_TABLE_NAME;
|
return TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME;
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
@ -1009,7 +1009,7 @@ public class TestMasterProcedureScheduler {
|
||||||
|
|
||||||
LockedResource tableResource = locks.get(1);
|
LockedResource tableResource = locks.get(1);
|
||||||
assertLockResource(tableResource, LockedResourceType.TABLE,
|
assertLockResource(tableResource, LockedResourceType.TABLE,
|
||||||
TableName.NAMESPACE_TABLE_NAME.getNameAsString());
|
TableProcedureInterface.DUMMY_NAMESPACE_TABLE_NAME.getNameAsString());
|
||||||
assertSharedLock(tableResource, 1);
|
assertSharedLock(tableResource, 1);
|
||||||
assertTrue(tableResource.getWaitingProcedures().isEmpty());
|
assertTrue(tableResource.getWaitingProcedures().isEmpty());
|
||||||
}
|
}
|
||||||
|
|
|
@ -240,11 +240,9 @@ public class TestModifyNamespaceProcedure {
|
||||||
ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, true);
|
ProcedureTestingUtility.setKillAndToggleBeforeStoreUpdate(procExec, true);
|
||||||
|
|
||||||
// Modify
|
// Modify
|
||||||
nsd.setConfiguration(nsKey, nsValue);
|
|
||||||
|
|
||||||
// Start the Modify procedure && kill the executor
|
// Start the Modify procedure && kill the executor
|
||||||
long procId = procExec.submitProcedure(
|
long procId = procExec.submitProcedure(new ModifyNamespaceProcedure(procExec.getEnvironment(),
|
||||||
new ModifyNamespaceProcedure(procExec.getEnvironment(), nsd));
|
NamespaceDescriptor.create(nsd).addConfiguration(nsKey, nsValue).build()));
|
||||||
|
|
||||||
int lastStep = 2; // failing before MODIFY_NAMESPACE_UPDATE_NS_TABLE
|
int lastStep = 2; // failing before MODIFY_NAMESPACE_UPDATE_NS_TABLE
|
||||||
MasterProcedureTestingUtility.testRollbackAndDoubleExecution(procExec, procId, lastStep);
|
MasterProcedureTestingUtility.testRollbackAndDoubleExecution(procExec, procId, lastStep);
|
||||||
|
|
|
@ -136,8 +136,6 @@ public class TestProcedurePriority {
|
||||||
.stream().filter(t -> !t.getRegionServer().getRegions(TableName.META_TABLE_NAME).isEmpty())
|
.stream().filter(t -> !t.getRegionServer().getRegions(TableName.META_TABLE_NAME).isEmpty())
|
||||||
.findAny().get();
|
.findAny().get();
|
||||||
HRegionServer rsNoMeta = UTIL.getOtherRegionServer(rsWithMetaThread.getRegionServer());
|
HRegionServer rsNoMeta = UTIL.getOtherRegionServer(rsWithMetaThread.getRegionServer());
|
||||||
// wait for NS table initialization to avoid our error inject affecting master initialization
|
|
||||||
UTIL.waitTableAvailable(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
FAIL = true;
|
FAIL = true;
|
||||||
UTIL.getMiniHBaseCluster().killRegionServer(rsNoMeta.getServerName());
|
UTIL.getMiniHBaseCluster().killRegionServer(rsNoMeta.getServerName());
|
||||||
// wait until all the worker thread are stuck, which means that the stuck checker will start to
|
// wait until all the worker thread are stuck, which means that the stuck checker will start to
|
||||||
|
|
|
@ -24,9 +24,11 @@ import static org.junit.Assert.assertTrue;
|
||||||
import org.apache.hadoop.conf.Configuration;
|
import org.apache.hadoop.conf.Configuration;
|
||||||
import org.apache.hadoop.hbase.HBaseClassTestRule;
|
import org.apache.hadoop.hbase.HBaseClassTestRule;
|
||||||
import org.apache.hadoop.hbase.HBaseTestingUtility;
|
import org.apache.hadoop.hbase.HBaseTestingUtility;
|
||||||
|
import org.apache.hadoop.hbase.TableName;
|
||||||
import org.apache.hadoop.hbase.regionserver.wal.FSHLog;
|
import org.apache.hadoop.hbase.regionserver.wal.FSHLog;
|
||||||
import org.apache.hadoop.hbase.testclassification.MediumTests;
|
import org.apache.hadoop.hbase.testclassification.MediumTests;
|
||||||
import org.apache.hadoop.hbase.testclassification.RegionServerTests;
|
import org.apache.hadoop.hbase.testclassification.RegionServerTests;
|
||||||
|
import org.apache.hadoop.hbase.util.Bytes;
|
||||||
import org.apache.hadoop.hbase.wal.AbstractFSWALProvider;
|
import org.apache.hadoop.hbase.wal.AbstractFSWALProvider;
|
||||||
import org.junit.After;
|
import org.junit.After;
|
||||||
import org.junit.Before;
|
import org.junit.Before;
|
||||||
|
@ -49,6 +51,9 @@ public class TestLogRoller {
|
||||||
public void setup() throws Exception {
|
public void setup() throws Exception {
|
||||||
TEST_UTIL.getConfiguration().setInt("hbase.regionserver.logroll.period", logRollPeriod);
|
TEST_UTIL.getConfiguration().setInt("hbase.regionserver.logroll.period", logRollPeriod);
|
||||||
TEST_UTIL.startMiniCluster(1);
|
TEST_UTIL.startMiniCluster(1);
|
||||||
|
TableName name = TableName.valueOf("Test");
|
||||||
|
TEST_UTIL.createTable(name, Bytes.toBytes("cf"));
|
||||||
|
TEST_UTIL.waitTableAvailable(name);
|
||||||
}
|
}
|
||||||
|
|
||||||
@After
|
@After
|
||||||
|
|
|
@ -443,7 +443,6 @@ public class TestPerColumnFamilyFlush {
|
||||||
* When a log roll is about to happen, we do a flush of the regions who will be affected by the
|
* When a log roll is about to happen, we do a flush of the regions who will be affected by the
|
||||||
* log roll. These flushes cannot be a selective flushes, otherwise we cannot roll the logs. This
|
* log roll. These flushes cannot be a selective flushes, otherwise we cannot roll the logs. This
|
||||||
* test ensures that we do a full-flush in that scenario.
|
* test ensures that we do a full-flush in that scenario.
|
||||||
* @throws IOException
|
|
||||||
*/
|
*/
|
||||||
@Test
|
@Test
|
||||||
public void testFlushingWhenLogRolling() throws Exception {
|
public void testFlushingWhenLogRolling() throws Exception {
|
||||||
|
@ -467,12 +466,6 @@ public class TestPerColumnFamilyFlush {
|
||||||
TEST_UTIL.startMiniCluster(numRegionServers);
|
TEST_UTIL.startMiniCluster(numRegionServers);
|
||||||
try {
|
try {
|
||||||
Table table = TEST_UTIL.createTable(tableName, FAMILIES);
|
Table table = TEST_UTIL.createTable(tableName, FAMILIES);
|
||||||
// Force flush the namespace table so edits to it are not hanging around as oldest
|
|
||||||
// edits. Otherwise, below, when we make maximum number of WAL files, then it will be
|
|
||||||
// the namespace region that is flushed and not the below 'desiredRegion'.
|
|
||||||
try (Admin admin = TEST_UTIL.getConnection().getAdmin()) {
|
|
||||||
admin.flush(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
}
|
|
||||||
Pair<HRegion, HRegionServer> desiredRegionAndServer = getRegionWithName(tableName);
|
Pair<HRegion, HRegionServer> desiredRegionAndServer = getRegionWithName(tableName);
|
||||||
final HRegion desiredRegion = desiredRegionAndServer.getFirst();
|
final HRegion desiredRegion = desiredRegionAndServer.getFirst();
|
||||||
assertTrue("Could not find a region which hosts the new region.", desiredRegion != null);
|
assertTrue("Could not find a region which hosts the new region.", desiredRegion != null);
|
||||||
|
|
|
@ -135,7 +135,7 @@ public class TestRegionReplicasWithRestartScenarios {
|
||||||
checkDuplicates(onlineRegions3);
|
checkDuplicates(onlineRegions3);
|
||||||
assertFalse(res);
|
assertFalse(res);
|
||||||
int totalRegions = onlineRegions.size() + onlineRegions2.size() + onlineRegions3.size();
|
int totalRegions = onlineRegions.size() + onlineRegions2.size() + onlineRegions3.size();
|
||||||
assertEquals(62, totalRegions);
|
assertEquals(61, totalRegions);
|
||||||
}
|
}
|
||||||
|
|
||||||
private boolean checkDuplicates(Collection<HRegion> onlineRegions3) throws Exception {
|
private boolean checkDuplicates(Collection<HRegion> onlineRegions3) throws Exception {
|
||||||
|
|
|
@ -239,7 +239,7 @@ public class TestRegionServerMetrics {
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
public void testRegionCount() throws Exception {
|
public void testRegionCount() throws Exception {
|
||||||
metricsHelper.assertGauge("regionCount", TABLES_ON_MASTER? 1: 3, serverSource);
|
metricsHelper.assertGauge("regionCount", TABLES_ON_MASTER ? 1 : 2, serverSource);
|
||||||
}
|
}
|
||||||
|
|
||||||
@Test
|
@Test
|
||||||
|
|
|
@ -254,9 +254,6 @@ public abstract class AbstractTestLogRolling {
|
||||||
final WAL log = server.getWAL(region.getRegionInfo());
|
final WAL log = server.getWAL(region.getRegionInfo());
|
||||||
Store s = region.getStore(HConstants.CATALOG_FAMILY);
|
Store s = region.getStore(HConstants.CATALOG_FAMILY);
|
||||||
|
|
||||||
//have to flush namespace to ensure it doesn't affect wall tests
|
|
||||||
admin.flush(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
|
|
||||||
// Put some stuff into table, to make sure we have some files to compact.
|
// Put some stuff into table, to make sure we have some files to compact.
|
||||||
for (int i = 1; i <= 2; ++i) {
|
for (int i = 1; i <= 2; ++i) {
|
||||||
doPut(table, i);
|
doPut(table, i);
|
||||||
|
|
|
@ -73,14 +73,7 @@ public class TestReplicationWALEntryFilters {
|
||||||
|
|
||||||
assertNull(filter.filter(metaEntry));
|
assertNull(filter.filter(metaEntry));
|
||||||
|
|
||||||
// ns table
|
|
||||||
WALKeyImpl key2 =
|
|
||||||
new WALKeyImpl(new byte[0], TableName.NAMESPACE_TABLE_NAME, System.currentTimeMillis());
|
|
||||||
Entry nsEntry = new Entry(key2, null);
|
|
||||||
assertNull(filter.filter(nsEntry));
|
|
||||||
|
|
||||||
// user table
|
// user table
|
||||||
|
|
||||||
WALKeyImpl key3 = new WALKeyImpl(new byte[0], TableName.valueOf("foo"),
|
WALKeyImpl key3 = new WALKeyImpl(new byte[0], TableName.valueOf("foo"),
|
||||||
System.currentTimeMillis());
|
System.currentTimeMillis());
|
||||||
Entry userEntry = new Entry(key3, null);
|
Entry userEntry = new Entry(key3, null);
|
||||||
|
|
|
@ -80,7 +80,6 @@ public class TestHBaseFsckMOB extends BaseTestHBaseFsck {
|
||||||
admin.setBalancerRunning(false, true);
|
admin.setBalancerRunning(false, true);
|
||||||
|
|
||||||
TEST_UTIL.waitUntilAllRegionsAssigned(TableName.META_TABLE_NAME);
|
TEST_UTIL.waitUntilAllRegionsAssigned(TableName.META_TABLE_NAME);
|
||||||
TEST_UTIL.waitUntilAllRegionsAssigned(TableName.NAMESPACE_TABLE_NAME);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@AfterClass
|
@AfterClass
|
||||||
|
|
Loading…
Reference in New Issue