Repository: hbase Updated Branches: refs/heads/branch-1.0 e1b947738 -> 7f1806dc3
HBASE-14361 ReplicationSink should create Connection instances lazily Project: http://git-wip-us.apache.org/repos/asf/hbase/repo Commit: http://git-wip-us.apache.org/repos/asf/hbase/commit/7f1806dc Tree: http://git-wip-us.apache.org/repos/asf/hbase/tree/7f1806dc Diff: http://git-wip-us.apache.org/repos/asf/hbase/diff/7f1806dc Branch: refs/heads/branch-1.0 Commit: 7f1806dc3f2f628dfa4473bfcd4283c28383d39b Parents: e1b9477 Author: stack <st...@apache.org> Authored: Tue Sep 15 05:15:47 2015 -0700 Committer: stack <st...@apache.org> Committed: Tue Sep 15 05:17:32 2015 -0700 ---------------------------------------------------------------------- .../regionserver/ReplicationSink.java | 27 +++++++++++++++++--- 1 file changed, 23 insertions(+), 4 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/hbase/blob/7f1806dc/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSink.java ---------------------------------------------------------------------- diff --git a/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSink.java b/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSink.java index 3276418..0a62c72 100644 --- a/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSink.java +++ b/hbase-server/src/main/java/org/apache/hadoop/hbase/replication/regionserver/ReplicationSink.java @@ -71,9 +71,12 @@ public class ReplicationSink { private static final Log LOG = LogFactory.getLog(ReplicationSink.class); private final Configuration conf; - private final Connection sharedHtableCon; + // Volatile because of note in here -- look for double-checked locking: + // http://www.oracle.com/technetwork/articles/javase/bloch-effective-08-qa-140880.html + private volatile Connection sharedHtableCon; private final MetricsSink metrics; private final AtomicLong totalReplicatedEdits = new AtomicLong(); + private final Object sharedHtableConLock = new Object(); /** * Create a sink for replication @@ -87,7 +90,6 @@ public class ReplicationSink { this.conf = HBaseConfiguration.create(conf); decorateConf(); this.metrics = new MetricsSink(); - this.sharedHtableCon = ConnectionFactory.createConnection(this.conf); } /** @@ -212,7 +214,14 @@ public class ReplicationSink { */ public void stopReplicationSinkServices() { try { - this.sharedHtableCon.close(); + if (this.sharedHtableCon != null) { + synchronized (sharedHtableConLock) { + if (this.sharedHtableCon != null) { + this.sharedHtableCon.close(); + this.sharedHtableCon = null; + } + } + } } catch (IOException e) { LOG.warn("IOException while closing the connection", e); // ignoring as we are closing. } @@ -231,7 +240,17 @@ public class ReplicationSink { } Table table = null; try { - table = this.sharedHtableCon.getTable(tableName); + // See https://en.wikipedia.org/wiki/Double-checked_locking + Connection connection = this.sharedHtableCon; + if (connection == null) { + synchronized (sharedHtableConLock) { + connection = this.sharedHtableCon; + if (connection == null) { + connection = this.sharedHtableCon = ConnectionFactory.createConnection(this.conf); + } + } + } + table = connection.getTable(tableName); for (List<Row> rows : allRows) { table.batch(rows); }