HBASE-21888 Add a isClosed method to AsyncConnection
Signed-off-by: Guanghao Zhang <zghao@apache.org>
This commit is contained in:
parent
cefaecdc83
commit
33bdf09f77
|
@ -21,7 +21,6 @@ import java.io.Closeable;
|
||||||
import java.io.IOException;
|
import java.io.IOException;
|
||||||
import java.util.concurrent.CompletableFuture;
|
import java.util.concurrent.CompletableFuture;
|
||||||
import java.util.concurrent.ExecutorService;
|
import java.util.concurrent.ExecutorService;
|
||||||
|
|
||||||
import org.apache.hadoop.conf.Configuration;
|
import org.apache.hadoop.conf.Configuration;
|
||||||
import org.apache.hadoop.hbase.HBaseInterfaceAudience;
|
import org.apache.hadoop.hbase.HBaseInterfaceAudience;
|
||||||
import org.apache.hadoop.hbase.ServerName;
|
import org.apache.hadoop.hbase.ServerName;
|
||||||
|
@ -196,6 +195,12 @@ public interface AsyncConnection extends Closeable {
|
||||||
*/
|
*/
|
||||||
AsyncBufferedMutatorBuilder getBufferedMutatorBuilder(TableName tableName, ExecutorService pool);
|
AsyncBufferedMutatorBuilder getBufferedMutatorBuilder(TableName tableName, ExecutorService pool);
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Returns whether the connection is closed or not.
|
||||||
|
* @return true if this connection is closed
|
||||||
|
*/
|
||||||
|
boolean isClosed();
|
||||||
|
|
||||||
/**
|
/**
|
||||||
* Retrieve an Hbck implementation to fix an HBase cluster. The returned Hbck is not guaranteed to
|
* Retrieve an Hbck implementation to fix an HBase cluster. The returned Hbck is not guaranteed to
|
||||||
* be thread-safe. A new instance should be created by each thread. This is a lightweight
|
* be thread-safe. A new instance should be created by each thread. This is a lightweight
|
||||||
|
|
|
@ -105,6 +105,8 @@ class AsyncConnectionImpl implements AsyncConnection {
|
||||||
|
|
||||||
private ChoreService authService;
|
private ChoreService authService;
|
||||||
|
|
||||||
|
private volatile boolean closed = false;
|
||||||
|
|
||||||
public AsyncConnectionImpl(Configuration conf, AsyncRegistry registry, String clusterId,
|
public AsyncConnectionImpl(Configuration conf, AsyncRegistry registry, String clusterId,
|
||||||
User user) {
|
User user) {
|
||||||
this.conf = conf;
|
this.conf = conf;
|
||||||
|
@ -140,11 +142,22 @@ class AsyncConnectionImpl implements AsyncConnection {
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
public void close() {
|
public void close() {
|
||||||
|
// As the code below is safe to be executed in parallel, here we do not use CAS or lock, just a
|
||||||
|
// simple volatile flag.
|
||||||
|
if (closed) {
|
||||||
|
return;
|
||||||
|
}
|
||||||
IOUtils.closeQuietly(rpcClient);
|
IOUtils.closeQuietly(rpcClient);
|
||||||
IOUtils.closeQuietly(registry);
|
IOUtils.closeQuietly(registry);
|
||||||
if (authService != null) {
|
if (authService != null) {
|
||||||
authService.shutdown();
|
authService.shutdown();
|
||||||
}
|
}
|
||||||
|
closed = true;
|
||||||
|
}
|
||||||
|
|
||||||
|
@Override
|
||||||
|
public boolean isClosed() {
|
||||||
|
return closed;
|
||||||
}
|
}
|
||||||
|
|
||||||
@Override
|
@Override
|
||||||
|
|
|
@ -111,11 +111,13 @@ public class TestAsyncTable {
|
||||||
TEST_UTIL.createTable(TABLE_NAME, FAMILY);
|
TEST_UTIL.createTable(TABLE_NAME, FAMILY);
|
||||||
TEST_UTIL.waitTableAvailable(TABLE_NAME);
|
TEST_UTIL.waitTableAvailable(TABLE_NAME);
|
||||||
ASYNC_CONN = ConnectionFactory.createAsyncConnection(TEST_UTIL.getConfiguration()).get();
|
ASYNC_CONN = ConnectionFactory.createAsyncConnection(TEST_UTIL.getConfiguration()).get();
|
||||||
|
assertFalse(ASYNC_CONN.isClosed());
|
||||||
}
|
}
|
||||||
|
|
||||||
@AfterClass
|
@AfterClass
|
||||||
public static void tearDownAfterClass() throws Exception {
|
public static void tearDownAfterClass() throws Exception {
|
||||||
IOUtils.closeQuietly(ASYNC_CONN);
|
IOUtils.closeQuietly(ASYNC_CONN);
|
||||||
|
assertTrue(ASYNC_CONN.isClosed());
|
||||||
TEST_UTIL.shutdownMiniCluster();
|
TEST_UTIL.shutdownMiniCluster();
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
Loading…
Reference in New Issue