Signed-off-by: Duo Zhang <zhangduo@apache.org>
This commit is contained in:
parent
5245e83c9c
commit
8e26761fd0
|
@ -49,7 +49,6 @@ import java.util.concurrent.Future;
|
||||||
import java.util.concurrent.TimeUnit;
|
import java.util.concurrent.TimeUnit;
|
||||||
import java.util.concurrent.TimeoutException;
|
import java.util.concurrent.TimeoutException;
|
||||||
import java.util.concurrent.atomic.AtomicInteger;
|
import java.util.concurrent.atomic.AtomicInteger;
|
||||||
import java.util.function.Function;
|
|
||||||
import java.util.regex.Pattern;
|
import java.util.regex.Pattern;
|
||||||
import java.util.stream.Collectors;
|
import java.util.stream.Collectors;
|
||||||
import javax.servlet.ServletException;
|
import javax.servlet.ServletException;
|
||||||
|
@ -686,10 +685,6 @@ public class HMaster extends HRegionServer implements MasterServices {
|
||||||
return connector.getLocalPort();
|
return connector.getLocalPort();
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
|
||||||
protected Function<TableDescriptorBuilder, TableDescriptorBuilder> getMetaTableObserver() {
|
|
||||||
return builder -> builder.setRegionReplication(conf.getInt(HConstants.META_REPLICAS_NUM, HConstants.DEFAULT_META_REPLICA_NUM));
|
|
||||||
}
|
|
||||||
/**
|
/**
|
||||||
* For compatibility, if failed with regionserver credentials, try the master one
|
* For compatibility, if failed with regionserver credentials, try the master one
|
||||||
*/
|
*/
|
||||||
|
|
|
@ -300,7 +300,7 @@ public class MasterFileSystem {
|
||||||
// assume, created table descriptor is for enabling table
|
// assume, created table descriptor is for enabling table
|
||||||
// meta table is a system table, so descriptors are predefined,
|
// meta table is a system table, so descriptors are predefined,
|
||||||
// we should get them from registry.
|
// we should get them from registry.
|
||||||
FSTableDescriptors fsd = new FSTableDescriptors(c, fs, rd);
|
FSTableDescriptors fsd = new FSTableDescriptors(fs, rd);
|
||||||
fsd.createTableDescriptor(fsd.get(TableName.META_TABLE_NAME));
|
fsd.createTableDescriptor(fsd.get(TableName.META_TABLE_NAME));
|
||||||
|
|
||||||
return rd;
|
return rd;
|
||||||
|
@ -406,6 +406,7 @@ public class MasterFileSystem {
|
||||||
// created here in bootstrap and it'll need to be cleaned up. Better to
|
// created here in bootstrap and it'll need to be cleaned up. Better to
|
||||||
// not make it in first place. Turn off block caching for bootstrap.
|
// not make it in first place. Turn off block caching for bootstrap.
|
||||||
// Enable after.
|
// Enable after.
|
||||||
|
FSTableDescriptors.tryUpdateMetaTableDescriptor(c);
|
||||||
TableDescriptor metaDescriptor = new FSTableDescriptors(c).get(TableName.META_TABLE_NAME);
|
TableDescriptor metaDescriptor = new FSTableDescriptors(c).get(TableName.META_TABLE_NAME);
|
||||||
HRegion meta = HRegion.createHRegion(RegionInfoBuilder.FIRST_META_REGIONINFO, rd,
|
HRegion meta = HRegion.createHRegion(RegionInfoBuilder.FIRST_META_REGIONINFO, rd,
|
||||||
c, setInfoFamilyCachingForMeta(metaDescriptor, false), null);
|
c, setInfoFamilyCachingForMeta(metaDescriptor, false), null);
|
||||||
|
|
|
@ -53,7 +53,6 @@ import java.util.concurrent.ConcurrentSkipListMap;
|
||||||
import java.util.concurrent.TimeUnit;
|
import java.util.concurrent.TimeUnit;
|
||||||
import java.util.concurrent.atomic.AtomicBoolean;
|
import java.util.concurrent.atomic.AtomicBoolean;
|
||||||
import java.util.concurrent.locks.ReentrantReadWriteLock;
|
import java.util.concurrent.locks.ReentrantReadWriteLock;
|
||||||
import java.util.function.Function;
|
|
||||||
import javax.management.MalformedObjectNameException;
|
import javax.management.MalformedObjectNameException;
|
||||||
import javax.servlet.http.HttpServlet;
|
import javax.servlet.http.HttpServlet;
|
||||||
import org.apache.commons.lang3.RandomUtils;
|
import org.apache.commons.lang3.RandomUtils;
|
||||||
|
@ -90,7 +89,6 @@ import org.apache.hadoop.hbase.client.ConnectionUtils;
|
||||||
import org.apache.hadoop.hbase.client.RegionInfo;
|
import org.apache.hadoop.hbase.client.RegionInfo;
|
||||||
import org.apache.hadoop.hbase.client.RegionInfoBuilder;
|
import org.apache.hadoop.hbase.client.RegionInfoBuilder;
|
||||||
import org.apache.hadoop.hbase.client.RpcRetryingCallerFactory;
|
import org.apache.hadoop.hbase.client.RpcRetryingCallerFactory;
|
||||||
import org.apache.hadoop.hbase.client.TableDescriptorBuilder;
|
|
||||||
import org.apache.hadoop.hbase.client.locking.EntityLock;
|
import org.apache.hadoop.hbase.client.locking.EntityLock;
|
||||||
import org.apache.hadoop.hbase.client.locking.LockServiceClient;
|
import org.apache.hadoop.hbase.client.locking.LockServiceClient;
|
||||||
import org.apache.hadoop.hbase.conf.ConfigurationManager;
|
import org.apache.hadoop.hbase.conf.ConfigurationManager;
|
||||||
|
@ -727,16 +725,13 @@ public class HRegionServer extends HasThread implements
|
||||||
FSUtils.setFsDefault(this.conf, FSUtils.getRootDir(this.conf));
|
FSUtils.setFsDefault(this.conf, FSUtils.getRootDir(this.conf));
|
||||||
this.dataFs = new HFileSystem(this.conf, useHBaseChecksum);
|
this.dataFs = new HFileSystem(this.conf, useHBaseChecksum);
|
||||||
this.dataRootDir = FSUtils.getRootDir(this.conf);
|
this.dataRootDir = FSUtils.getRootDir(this.conf);
|
||||||
this.tableDescriptors = getFsTableDescriptors();
|
this.tableDescriptors =
|
||||||
}
|
new FSTableDescriptors(this.dataFs, this.dataRootDir, !canUpdateTableDescriptor(), false);
|
||||||
|
if (this instanceof HMaster) {
|
||||||
private TableDescriptors getFsTableDescriptors() throws IOException {
|
FSTableDescriptors.tryUpdateMetaTableDescriptor(this.conf, this.dataFs, this.dataRootDir,
|
||||||
return new FSTableDescriptors(this.conf,
|
builder -> builder.setRegionReplication(
|
||||||
this.dataFs, this.dataRootDir, !canUpdateTableDescriptor(), false, getMetaTableObserver());
|
conf.getInt(HConstants.META_REPLICAS_NUM, HConstants.DEFAULT_META_REPLICA_NUM)));
|
||||||
}
|
}
|
||||||
|
|
||||||
protected Function<TableDescriptorBuilder, TableDescriptorBuilder> getMetaTableObserver() {
|
|
||||||
return null;
|
|
||||||
}
|
}
|
||||||
|
|
||||||
protected void login(UserProvider user, String host) throws IOException {
|
protected void login(UserProvider user, String host) throws IOException {
|
||||||
|
@ -744,7 +739,6 @@ public class HRegionServer extends HasThread implements
|
||||||
SecurityConstants.REGIONSERVER_KRB_PRINCIPAL, host);
|
SecurityConstants.REGIONSERVER_KRB_PRINCIPAL, host);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Wait for an active Master.
|
* Wait for an active Master.
|
||||||
* See override in Master superclass for how it is used.
|
* See override in Master superclass for how it is used.
|
||||||
|
|
|
@ -100,56 +100,49 @@ public class FSTableDescriptors implements TableDescriptors {
|
||||||
private final Map<TableName, TableDescriptor> cache = new ConcurrentHashMap<>();
|
private final Map<TableName, TableDescriptor> cache = new ConcurrentHashMap<>();
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Construct a FSTableDescriptors instance using the hbase root dir of the given
|
* Construct a FSTableDescriptors instance using the hbase root dir of the given conf and the
|
||||||
* conf and the filesystem where that root dir lives.
|
* filesystem where that root dir lives. This instance can do write operations (is not read only).
|
||||||
* This instance can do write operations (is not read only).
|
|
||||||
*/
|
*/
|
||||||
public FSTableDescriptors(final Configuration conf) throws IOException {
|
public FSTableDescriptors(final Configuration conf) throws IOException {
|
||||||
this(conf, FSUtils.getCurrentFileSystem(conf), FSUtils.getRootDir(conf));
|
this(FSUtils.getCurrentFileSystem(conf), FSUtils.getRootDir(conf));
|
||||||
}
|
}
|
||||||
|
|
||||||
public FSTableDescriptors(final Configuration conf, final FileSystem fs, final Path rootdir)
|
public FSTableDescriptors(final FileSystem fs, final Path rootdir) {
|
||||||
throws IOException {
|
this(fs, rootdir, false, true);
|
||||||
this(conf, fs, rootdir, false, true);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
public FSTableDescriptors(final FileSystem fs, final Path rootdir, final boolean fsreadonly,
|
||||||
* @param fsreadonly True if we are read-only when it comes to filesystem
|
final boolean usecache) {
|
||||||
* operations; i.e. on remove, we do not do delete in fs.
|
|
||||||
*/
|
|
||||||
@VisibleForTesting
|
|
||||||
public FSTableDescriptors(final Configuration conf, final FileSystem fs,
|
|
||||||
final Path rootdir, final boolean fsreadonly, final boolean usecache) throws IOException {
|
|
||||||
this(conf, fs, rootdir, fsreadonly, usecache, null);
|
|
||||||
}
|
|
||||||
|
|
||||||
/**
|
|
||||||
* @param fsreadonly True if we are read-only when it comes to filesystem
|
|
||||||
* operations; i.e. on remove, we do not do delete in fs.
|
|
||||||
* @param metaObserver Used by HMaster. It need to modify the META_REPLICAS_NUM for meta table descriptor.
|
|
||||||
* see HMaster#finishActiveMasterInitialization
|
|
||||||
* TODO: This is a workaround. Should remove this ugly code...
|
|
||||||
*/
|
|
||||||
public FSTableDescriptors(final Configuration conf, final FileSystem fs, final Path rootdir,
|
|
||||||
final boolean fsreadonly, final boolean usecache,
|
|
||||||
Function<TableDescriptorBuilder, TableDescriptorBuilder> metaObserver) throws IOException {
|
|
||||||
this.fs = fs;
|
this.fs = fs;
|
||||||
this.rootdir = rootdir;
|
this.rootdir = rootdir;
|
||||||
this.fsreadonly = fsreadonly;
|
this.fsreadonly = fsreadonly;
|
||||||
this.usecache = usecache;
|
this.usecache = usecache;
|
||||||
if (!fsreadonly) {
|
}
|
||||||
// see if we already have meta descriptor on fs. Write one if not.
|
|
||||||
try {
|
public static void tryUpdateMetaTableDescriptor(Configuration conf) throws IOException {
|
||||||
getTableDescriptorFromFs(fs, rootdir, TableName.META_TABLE_NAME);
|
tryUpdateMetaTableDescriptor(conf, FSUtils.getCurrentFileSystem(conf), FSUtils.getRootDir(conf),
|
||||||
} catch (TableInfoMissingException e) {
|
null);
|
||||||
TableDescriptorBuilder builder = createMetaTableDescriptorBuilder(conf);
|
}
|
||||||
if (metaObserver != null) {
|
|
||||||
builder = metaObserver.apply(builder);
|
public static void tryUpdateMetaTableDescriptor(Configuration conf, FileSystem fs, Path rootdir,
|
||||||
}
|
Function<TableDescriptorBuilder, TableDescriptorBuilder> metaObserver) throws IOException {
|
||||||
TableDescriptor td = builder.build();
|
// see if we already have meta descriptor on fs. Write one if not.
|
||||||
LOG.info("Creating new hbase:meta table default descriptor/schema {}", td);
|
try {
|
||||||
updateTableDescriptor(td);
|
getTableDescriptorFromFs(fs, rootdir, TableName.META_TABLE_NAME);
|
||||||
|
} catch (TableInfoMissingException e) {
|
||||||
|
TableDescriptorBuilder builder = createMetaTableDescriptorBuilder(conf);
|
||||||
|
if (metaObserver != null) {
|
||||||
|
builder = metaObserver.apply(builder);
|
||||||
}
|
}
|
||||||
|
TableDescriptor td = builder.build();
|
||||||
|
LOG.info("Creating new hbase:meta table descriptor {}", td);
|
||||||
|
TableName tableName = td.getTableName();
|
||||||
|
Path tableDir = FSUtils.getTableDir(rootdir, tableName);
|
||||||
|
Path p = writeTableDescriptor(fs, td, tableDir, getTableInfoPath(fs, tableDir, true));
|
||||||
|
if (p == null) {
|
||||||
|
throw new IOException("Failed update hbase:meta table descriptor");
|
||||||
|
}
|
||||||
|
LOG.info("Updated hbase:meta table descriptor to {}", p);
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -324,8 +317,7 @@ public class FSTableDescriptors implements TableDescriptors {
|
||||||
* from the FileSystem.
|
* from the FileSystem.
|
||||||
*/
|
*/
|
||||||
@Override
|
@Override
|
||||||
public TableDescriptor remove(final TableName tablename)
|
public TableDescriptor remove(final TableName tablename) throws IOException {
|
||||||
throws IOException {
|
|
||||||
if (fsreadonly) {
|
if (fsreadonly) {
|
||||||
throw new NotImplementedException("Cannot remove a table descriptor - in read only mode");
|
throw new NotImplementedException("Cannot remove a table descriptor - in read only mode");
|
||||||
}
|
}
|
||||||
|
@ -339,8 +331,7 @@ public class FSTableDescriptors implements TableDescriptors {
|
||||||
return descriptor;
|
return descriptor;
|
||||||
}
|
}
|
||||||
|
|
||||||
private FileStatus getTableInfoPath(Path tableDir)
|
private FileStatus getTableInfoPath(Path tableDir) throws IOException {
|
||||||
throws IOException {
|
|
||||||
return getTableInfoPath(fs, tableDir, !fsreadonly);
|
return getTableInfoPath(fs, tableDir, !fsreadonly);
|
||||||
}
|
}
|
||||||
|
|
||||||
|
@ -372,7 +363,7 @@ public class FSTableDescriptors implements TableDescriptors {
|
||||||
* @return The file status of the current table info file or null if none exist
|
* @return The file status of the current table info file or null if none exist
|
||||||
*/
|
*/
|
||||||
private static FileStatus getTableInfoPath(FileSystem fs, Path tableDir, boolean removeOldFiles)
|
private static FileStatus getTableInfoPath(FileSystem fs, Path tableDir, boolean removeOldFiles)
|
||||||
throws IOException {
|
throws IOException {
|
||||||
Path tableInfoDir = new Path(tableDir, TABLEINFO_DIR);
|
Path tableInfoDir = new Path(tableDir, TABLEINFO_DIR);
|
||||||
return getCurrentTableInfoStatus(fs, tableInfoDir, removeOldFiles);
|
return getCurrentTableInfoStatus(fs, tableInfoDir, removeOldFiles);
|
||||||
}
|
}
|
||||||
|
@ -711,8 +702,8 @@ public class FSTableDescriptors implements TableDescriptors {
|
||||||
* already exists and we weren't forcing the descriptor creation.
|
* already exists and we weren't forcing the descriptor creation.
|
||||||
* @throws IOException if a filesystem error occurs
|
* @throws IOException if a filesystem error occurs
|
||||||
*/
|
*/
|
||||||
public static boolean createTableDescriptorForTableDirectory(FileSystem fs,
|
public static boolean createTableDescriptorForTableDirectory(FileSystem fs, Path tableDir,
|
||||||
Path tableDir, TableDescriptor htd, boolean forceCreation) throws IOException {
|
TableDescriptor htd, boolean forceCreation) throws IOException {
|
||||||
FileStatus status = getTableInfoPath(fs, tableDir);
|
FileStatus status = getTableInfoPath(fs, tableDir);
|
||||||
if (status != null) {
|
if (status != null) {
|
||||||
LOG.debug("Current path=" + status.getPath());
|
LOG.debug("Current path=" + status.getPath());
|
||||||
|
|
|
@ -52,7 +52,7 @@ public class TestFSTableDescriptorForceCreation {
|
||||||
final String name = this.name.getMethodName();
|
final String name = this.name.getMethodName();
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, rootdir);
|
||||||
|
|
||||||
assertTrue("Should create new table descriptor",
|
assertTrue("Should create new table descriptor",
|
||||||
fstd.createTableDescriptor(TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build(), false));
|
fstd.createTableDescriptor(TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build(), false));
|
||||||
|
@ -65,7 +65,7 @@ public class TestFSTableDescriptorForceCreation {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any detritus laying around.
|
// Cleanup old tests if any detritus laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, rootdir);
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
||||||
fstd.update(htd);
|
fstd.update(htd);
|
||||||
assertFalse("Should not create new table descriptor", fstd.createTableDescriptor(htd, false));
|
assertFalse("Should not create new table descriptor", fstd.createTableDescriptor(htd, false));
|
||||||
|
@ -77,7 +77,7 @@ public class TestFSTableDescriptorForceCreation {
|
||||||
final String name = this.name.getMethodName();
|
final String name = this.name.getMethodName();
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, rootdir);
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
||||||
fstd.createTableDescriptor(htd, false);
|
fstd.createTableDescriptor(htd, false);
|
||||||
assertTrue("Should create new table descriptor",
|
assertTrue("Should create new table descriptor",
|
||||||
|
|
|
@ -43,7 +43,6 @@ import org.junit.rules.TestName;
|
||||||
*/
|
*/
|
||||||
@Category({CoprocessorTests.class, SmallTests.class})
|
@Category({CoprocessorTests.class, SmallTests.class})
|
||||||
public class TestCoreMasterCoprocessor {
|
public class TestCoreMasterCoprocessor {
|
||||||
|
|
||||||
@ClassRule
|
@ClassRule
|
||||||
public static final HBaseClassTestRule CLASS_RULE =
|
public static final HBaseClassTestRule CLASS_RULE =
|
||||||
HBaseClassTestRule.forClass(TestCoreMasterCoprocessor.class);
|
HBaseClassTestRule.forClass(TestCoreMasterCoprocessor.class);
|
||||||
|
|
|
@ -89,7 +89,6 @@ public class TestGetClosestAtOrBefore {
|
||||||
// Up flush size else we bind up when we use default catalog flush of 16k.
|
// Up flush size else we bind up when we use default catalog flush of 16k.
|
||||||
TableDescriptorBuilder metaBuilder = UTIL.getMetaTableDescriptorBuilder()
|
TableDescriptorBuilder metaBuilder = UTIL.getMetaTableDescriptorBuilder()
|
||||||
.setMemStoreFlushSize(64 * 1024 * 1024);
|
.setMemStoreFlushSize(64 * 1024 * 1024);
|
||||||
|
|
||||||
HRegion mr = HBaseTestingUtility.createRegionAndWAL(HRegionInfo.FIRST_META_REGIONINFO,
|
HRegion mr = HBaseTestingUtility.createRegionAndWAL(HRegionInfo.FIRST_META_REGIONINFO,
|
||||||
rootdir, this.conf, metaBuilder.build());
|
rootdir, this.conf, metaBuilder.build());
|
||||||
try {
|
try {
|
||||||
|
|
|
@ -146,6 +146,7 @@ public class TestHRegionInfo {
|
||||||
Path basedir = htu.getDataTestDir();
|
Path basedir = htu.getDataTestDir();
|
||||||
// Create a region. That'll write the .regioninfo file.
|
// Create a region. That'll write the .regioninfo file.
|
||||||
FSTableDescriptors fsTableDescriptors = new FSTableDescriptors(htu.getConfiguration());
|
FSTableDescriptors fsTableDescriptors = new FSTableDescriptors(htu.getConfiguration());
|
||||||
|
FSTableDescriptors.tryUpdateMetaTableDescriptor(htu.getConfiguration());
|
||||||
HRegion r = HBaseTestingUtility.createRegionAndWAL(hri, basedir, htu.getConfiguration(),
|
HRegion r = HBaseTestingUtility.createRegionAndWAL(hri, basedir, htu.getConfiguration(),
|
||||||
fsTableDescriptors.get(TableName.META_TABLE_NAME));
|
fsTableDescriptors.get(TableName.META_TABLE_NAME));
|
||||||
// Get modtime on the file.
|
// Get modtime on the file.
|
||||||
|
|
|
@ -103,6 +103,7 @@ public class TestRegionInfoBuilder {
|
||||||
Path basedir = htu.getDataTestDir();
|
Path basedir = htu.getDataTestDir();
|
||||||
// Create a region. That'll write the .regioninfo file.
|
// Create a region. That'll write the .regioninfo file.
|
||||||
FSTableDescriptors fsTableDescriptors = new FSTableDescriptors(htu.getConfiguration());
|
FSTableDescriptors fsTableDescriptors = new FSTableDescriptors(htu.getConfiguration());
|
||||||
|
FSTableDescriptors.tryUpdateMetaTableDescriptor(htu.getConfiguration());
|
||||||
HRegion r = HBaseTestingUtility.createRegionAndWAL(convert(ri), basedir, htu.getConfiguration(),
|
HRegion r = HBaseTestingUtility.createRegionAndWAL(convert(ri), basedir, htu.getConfiguration(),
|
||||||
fsTableDescriptors.get(TableName.META_TABLE_NAME));
|
fsTableDescriptors.get(TableName.META_TABLE_NAME));
|
||||||
// Get modtime on the file.
|
// Get modtime on the file.
|
||||||
|
|
|
@ -91,8 +91,8 @@ public class TestLogRollingNoCluster {
|
||||||
final Configuration conf = new Configuration(TEST_UTIL.getConfiguration());
|
final Configuration conf = new Configuration(TEST_UTIL.getConfiguration());
|
||||||
conf.set(WALFactory.WAL_PROVIDER, "filesystem");
|
conf.set(WALFactory.WAL_PROVIDER, "filesystem");
|
||||||
FSUtils.setRootDir(conf, dir);
|
FSUtils.setRootDir(conf, dir);
|
||||||
FSTableDescriptors fsTableDescriptors =
|
FSTableDescriptors fsTableDescriptors = new FSTableDescriptors(TEST_UTIL.getConfiguration());
|
||||||
new FSTableDescriptors(TEST_UTIL.getConfiguration());
|
FSTableDescriptors.tryUpdateMetaTableDescriptor(TEST_UTIL.getConfiguration());
|
||||||
TableDescriptor metaTableDescriptor = fsTableDescriptors.get(TableName.META_TABLE_NAME);
|
TableDescriptor metaTableDescriptor = fsTableDescriptors.get(TableName.META_TABLE_NAME);
|
||||||
conf.set("hbase.regionserver.hlog.writer.impl", HighLatencySyncWriter.class.getName());
|
conf.set("hbase.regionserver.hlog.writer.impl", HighLatencySyncWriter.class.getName());
|
||||||
final WALFactory wals = new WALFactory(conf, TestLogRollingNoCluster.class.getName());
|
final WALFactory wals = new WALFactory(conf, TestLogRollingNoCluster.class.getName());
|
||||||
|
|
|
@ -420,8 +420,8 @@ public class TestRegionReplicaReplicationEndpoint {
|
||||||
RegionReplicaReplicationEndpoint.RegionReplicaOutputSink sink =
|
RegionReplicaReplicationEndpoint.RegionReplicaOutputSink sink =
|
||||||
mock(RegionReplicaReplicationEndpoint.RegionReplicaOutputSink.class);
|
mock(RegionReplicaReplicationEndpoint.RegionReplicaOutputSink.class);
|
||||||
when(sink.getSkippedEditsCounter()).thenReturn(skippedEdits);
|
when(sink.getSkippedEditsCounter()).thenReturn(skippedEdits);
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(HTU.getConfiguration(),
|
FSTableDescriptors fstd =
|
||||||
FileSystem.get(HTU.getConfiguration()), HTU.getDefaultRootDirPath());
|
new FSTableDescriptors(FileSystem.get(HTU.getConfiguration()), HTU.getDefaultRootDirPath());
|
||||||
RegionReplicaReplicationEndpoint.RegionReplicaSinkWriter sinkWriter =
|
RegionReplicaReplicationEndpoint.RegionReplicaSinkWriter sinkWriter =
|
||||||
new RegionReplicaReplicationEndpoint.RegionReplicaSinkWriter(sink,
|
new RegionReplicaReplicationEndpoint.RegionReplicaSinkWriter(sink,
|
||||||
(ClusterConnection) connection, Executors.newSingleThreadExecutor(), Integer.MAX_VALUE,
|
(ClusterConnection) connection, Executors.newSingleThreadExecutor(), Integer.MAX_VALUE,
|
||||||
|
|
|
@ -29,7 +29,6 @@ import java.io.IOException;
|
||||||
import java.util.Arrays;
|
import java.util.Arrays;
|
||||||
import java.util.Comparator;
|
import java.util.Comparator;
|
||||||
import java.util.Map;
|
import java.util.Map;
|
||||||
import org.apache.hadoop.conf.Configuration;
|
|
||||||
import org.apache.hadoop.fs.FSDataInputStream;
|
import org.apache.hadoop.fs.FSDataInputStream;
|
||||||
import org.apache.hadoop.fs.FSDataOutputStream;
|
import org.apache.hadoop.fs.FSDataOutputStream;
|
||||||
import org.apache.hadoop.fs.FileStatus;
|
import org.apache.hadoop.fs.FileStatus;
|
||||||
|
@ -87,7 +86,7 @@ public class TestFSTableDescriptors {
|
||||||
Path testdir = UTIL.getDataTestDir(name.getMethodName());
|
Path testdir = UTIL.getDataTestDir(name.getMethodName());
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, testdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, testdir);
|
||||||
assertTrue(fstd.createTableDescriptor(htd));
|
assertTrue(fstd.createTableDescriptor(htd));
|
||||||
assertFalse(fstd.createTableDescriptor(htd));
|
assertFalse(fstd.createTableDescriptor(htd));
|
||||||
FileStatus [] statuses = fs.listStatus(testdir);
|
FileStatus [] statuses = fs.listStatus(testdir);
|
||||||
|
@ -107,7 +106,7 @@ public class TestFSTableDescriptors {
|
||||||
Path testdir = UTIL.getDataTestDir(name.getMethodName());
|
Path testdir = UTIL.getDataTestDir(name.getMethodName());
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, testdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, testdir);
|
||||||
Path p0 = fstd.updateTableDescriptor(htd);
|
Path p0 = fstd.updateTableDescriptor(htd);
|
||||||
int i0 = FSTableDescriptors.getTableInfoSequenceId(p0);
|
int i0 = FSTableDescriptors.getTableInfoSequenceId(p0);
|
||||||
Path p1 = fstd.updateTableDescriptor(htd);
|
Path p1 = fstd.updateTableDescriptor(htd);
|
||||||
|
@ -167,7 +166,7 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any detrius laying around.
|
// Cleanup old tests if any detrius laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
TableDescriptors htds = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
TableDescriptors htds = new FSTableDescriptors(fs, rootdir);
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
||||||
htds.update(htd);
|
htds.update(htd);
|
||||||
assertNotNull(htds.remove(htd.getTableName()));
|
assertNotNull(htds.remove(htd.getTableName()));
|
||||||
|
@ -179,7 +178,7 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
||||||
Path rootdir = UTIL.getDataTestDir(name);
|
Path rootdir = UTIL.getDataTestDir(name);
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, rootdir);
|
||||||
fstd.createTableDescriptor(htd);
|
fstd.createTableDescriptor(htd);
|
||||||
TableDescriptor td2 =
|
TableDescriptor td2 =
|
||||||
FSTableDescriptors.getTableDescriptorFromFs(fs, rootdir, htd.getTableName());
|
FSTableDescriptors.getTableDescriptorFromFs(fs, rootdir, htd.getTableName());
|
||||||
|
@ -190,13 +189,13 @@ public class TestFSTableDescriptors {
|
||||||
final String name = this.name.getMethodName();
|
final String name = this.name.getMethodName();
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
Path rootdir = UTIL.getDataTestDir(name);
|
Path rootdir = UTIL.getDataTestDir(name);
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, rootdir);
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
||||||
Path descriptorFile = fstd.updateTableDescriptor(htd);
|
Path descriptorFile = fstd.updateTableDescriptor(htd);
|
||||||
try (FSDataOutputStream out = fs.create(descriptorFile, true)) {
|
try (FSDataOutputStream out = fs.create(descriptorFile, true)) {
|
||||||
out.write(TableDescriptorBuilder.toByteArray(htd));
|
out.write(TableDescriptorBuilder.toByteArray(htd));
|
||||||
}
|
}
|
||||||
FSTableDescriptors fstd2 = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors fstd2 = new FSTableDescriptors(fs, rootdir);
|
||||||
TableDescriptor td2 = fstd2.get(htd.getTableName());
|
TableDescriptor td2 = fstd2.get(htd.getTableName());
|
||||||
assertEquals(htd, td2);
|
assertEquals(htd, td2);
|
||||||
FileStatus descriptorFile2 =
|
FileStatus descriptorFile2 =
|
||||||
|
@ -215,7 +214,7 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any debris laying around.
|
// Cleanup old tests if any debris laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors htds = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir) {
|
FSTableDescriptors htds = new FSTableDescriptors(fs, rootdir) {
|
||||||
@Override
|
@Override
|
||||||
public TableDescriptor get(TableName tablename)
|
public TableDescriptor get(TableName tablename)
|
||||||
throws TableExistsException, FileNotFoundException, IOException {
|
throws TableExistsException, FileNotFoundException, IOException {
|
||||||
|
@ -261,8 +260,7 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any debris laying around.
|
// Cleanup old tests if any debris laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors htds = new FSTableDescriptorsTest(UTIL.getConfiguration(), fs, rootdir,
|
FSTableDescriptors htds = new FSTableDescriptorsTest(fs, rootdir, false);
|
||||||
false, false);
|
|
||||||
final int count = 10;
|
final int count = 10;
|
||||||
// Write out table infos.
|
// Write out table infos.
|
||||||
for (int i = 0; i < count; i++) {
|
for (int i = 0; i < count; i++) {
|
||||||
|
@ -294,7 +292,7 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any debris laying around.
|
// Cleanup old tests if any debris laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors htds = new FSTableDescriptorsTest(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors htds = new FSTableDescriptorsTest(fs, rootdir);
|
||||||
final int count = 4;
|
final int count = 4;
|
||||||
// Write out table infos.
|
// Write out table infos.
|
||||||
for (int i = 0; i < count; i++) {
|
for (int i = 0; i < count; i++) {
|
||||||
|
@ -313,7 +311,7 @@ public class TestFSTableDescriptors {
|
||||||
final String name = "testGetAllOrdering";
|
final String name = "testGetAllOrdering";
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
Path rootDir = new Path(UTIL.getDataTestDir(), name);
|
Path rootDir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors tds = new FSTableDescriptorsTest(UTIL.getConfiguration(), fs, rootDir);
|
FSTableDescriptors tds = new FSTableDescriptorsTest(fs, rootDir);
|
||||||
|
|
||||||
String[] tableNames = new String[] { "foo", "bar", "foo:bar", "bar:foo" };
|
String[] tableNames = new String[] { "foo", "bar", "foo:bar", "bar:foo" };
|
||||||
for (String tableName : tableNames) {
|
for (String tableName : tableNames) {
|
||||||
|
@ -347,9 +345,8 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any debris laying around.
|
// Cleanup old tests if any debris laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
FSTableDescriptors chtds = new FSTableDescriptorsTest(UTIL.getConfiguration(), fs, rootdir);
|
FSTableDescriptors chtds = new FSTableDescriptorsTest(fs, rootdir);
|
||||||
FSTableDescriptors nonchtds = new FSTableDescriptorsTest(UTIL.getConfiguration(), fs,
|
FSTableDescriptors nonchtds = new FSTableDescriptorsTest(fs, rootdir, false);
|
||||||
rootdir, false, false);
|
|
||||||
|
|
||||||
final int count = 10;
|
final int count = 10;
|
||||||
// Write out table infos via non-cached FSTableDescriptors
|
// Write out table infos via non-cached FSTableDescriptors
|
||||||
|
@ -387,7 +384,7 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any detrius laying around.
|
// Cleanup old tests if any detrius laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
TableDescriptors htds = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
TableDescriptors htds = new FSTableDescriptors(fs, rootdir);
|
||||||
assertNull("There shouldn't be any HTD for this table",
|
assertNull("There shouldn't be any HTD for this table",
|
||||||
htds.get(TableName.valueOf("NoSuchTable")));
|
htds.get(TableName.valueOf("NoSuchTable")));
|
||||||
}
|
}
|
||||||
|
@ -398,7 +395,7 @@ public class TestFSTableDescriptors {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
// Cleanup old tests if any detrius laying around.
|
// Cleanup old tests if any detrius laying around.
|
||||||
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
Path rootdir = new Path(UTIL.getDataTestDir(), name);
|
||||||
TableDescriptors htds = new FSTableDescriptors(UTIL.getConfiguration(), fs, rootdir);
|
TableDescriptors htds = new FSTableDescriptors(fs, rootdir);
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name)).build();
|
||||||
htds.update(htd);
|
htds.update(htd);
|
||||||
htds.update(htd);
|
htds.update(htd);
|
||||||
|
@ -435,8 +432,7 @@ public class TestFSTableDescriptors {
|
||||||
public void testReadingInvalidDirectoryFromFS() throws IOException {
|
public void testReadingInvalidDirectoryFromFS() throws IOException {
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
try {
|
try {
|
||||||
new FSTableDescriptors(UTIL.getConfiguration(), fs,
|
new FSTableDescriptors(fs, FSUtils.getRootDir(UTIL.getConfiguration()))
|
||||||
FSUtils.getRootDir(UTIL.getConfiguration()))
|
|
||||||
.get(TableName.valueOf(HConstants.HBASE_TEMP_DIRECTORY));
|
.get(TableName.valueOf(HConstants.HBASE_TEMP_DIRECTORY));
|
||||||
fail("Shouldn't be able to read a table descriptor for the archive directory.");
|
fail("Shouldn't be able to read a table descriptor for the archive directory.");
|
||||||
} catch (Exception e) {
|
} catch (Exception e) {
|
||||||
|
@ -450,7 +446,7 @@ public class TestFSTableDescriptors {
|
||||||
Path testdir = UTIL.getDataTestDir(name.getMethodName());
|
Path testdir = UTIL.getDataTestDir(name.getMethodName());
|
||||||
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
|
TableDescriptor htd = TableDescriptorBuilder.newBuilder(TableName.valueOf(name.getMethodName())).build();
|
||||||
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
FileSystem fs = FileSystem.get(UTIL.getConfiguration());
|
||||||
FSTableDescriptors fstd = new FSTableDescriptors(UTIL.getConfiguration(), fs, testdir);
|
FSTableDescriptors fstd = new FSTableDescriptors(fs, testdir);
|
||||||
assertTrue(fstd.createTableDescriptor(htd));
|
assertTrue(fstd.createTableDescriptor(htd));
|
||||||
assertFalse(fstd.createTableDescriptor(htd));
|
assertFalse(fstd.createTableDescriptor(htd));
|
||||||
htd = TableDescriptorBuilder.newBuilder(htd)
|
htd = TableDescriptorBuilder.newBuilder(htd)
|
||||||
|
@ -467,14 +463,12 @@ public class TestFSTableDescriptors {
|
||||||
|
|
||||||
private static class FSTableDescriptorsTest extends FSTableDescriptors {
|
private static class FSTableDescriptorsTest extends FSTableDescriptors {
|
||||||
|
|
||||||
public FSTableDescriptorsTest(Configuration conf, FileSystem fs, Path rootdir)
|
public FSTableDescriptorsTest(FileSystem fs, Path rootdir) {
|
||||||
throws IOException {
|
this(fs, rootdir, true);
|
||||||
this(conf, fs, rootdir, false, true);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
public FSTableDescriptorsTest(Configuration conf, FileSystem fs, Path rootdir,
|
public FSTableDescriptorsTest(FileSystem fs, Path rootdir, boolean usecache) {
|
||||||
boolean fsreadonly, boolean usecache) throws IOException {
|
super(fs, rootdir, false, usecache);
|
||||||
super(conf, fs, rootdir, fsreadonly, usecache);
|
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|
Loading…
Reference in New Issue