Return-Path: X-Original-To: archive-asf-public-internal@cust-asf2.ponee.io Delivered-To: archive-asf-public-internal@cust-asf2.ponee.io Received: from cust-asf.ponee.io (cust-asf.ponee.io [163.172.22.183]) by cust-asf2.ponee.io (Postfix) with ESMTP id 10AC4200BA6 for ; Sat, 10 Sep 2016 01:37:24 +0200 (CEST) Received: by cust-asf.ponee.io (Postfix) id 0F9EE160ADB; Fri, 9 Sep 2016 23:37:24 +0000 (UTC) Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [140.211.11.3]) by cust-asf.ponee.io (Postfix) with SMTP id A3431160AE3 for ; Sat, 10 Sep 2016 01:37:22 +0200 (CEST) Received: (qmail 29478 invoked by uid 500); 9 Sep 2016 23:37:21 -0000 Mailing-List: contact commits-help@accumulo.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@accumulo.apache.org Delivered-To: mailing list commits@accumulo.apache.org Received: (qmail 28956 invoked by uid 99); 9 Sep 2016 23:37:21 -0000 Received: from git1-us-west.apache.org (HELO git1-us-west.apache.org) (140.211.11.23) by apache.org (qpsmtpd/0.29) with ESMTP; Fri, 09 Sep 2016 23:37:21 +0000 Received: by git1-us-west.apache.org (ASF Mail Server at git1-us-west.apache.org, from userid 33) id F3FB5E094C; Fri, 9 Sep 2016 23:37:20 +0000 (UTC) Content-Type: text/plain; charset="us-ascii" MIME-Version: 1.0 Content-Transfer-Encoding: 7bit From: ctubbsii@apache.org To: commits@accumulo.apache.org Date: Fri, 09 Sep 2016 23:37:31 -0000 Message-Id: In-Reply-To: References: X-Mailer: ASF-Git Admin Mailer Subject: [12/45] accumulo git commit: ACCUMULO-4391 Use a closed flag archived-at: Fri, 09 Sep 2016 23:37:24 -0000 ACCUMULO-4391 Use a closed flag Project: http://git-wip-us.apache.org/repos/asf/accumulo/repo Commit: http://git-wip-us.apache.org/repos/asf/accumulo/commit/9038d8b3 Tree: http://git-wip-us.apache.org/repos/asf/accumulo/tree/9038d8b3 Diff: http://git-wip-us.apache.org/repos/asf/accumulo/diff/9038d8b3 Branch: refs/heads/1.7 Commit: 9038d8b3a36e7189fbdedfbbb950c7b1f43b2b7d Parents: 6ff35f0 Author: Ivan Bella Authored: Thu Aug 4 00:04:26 2016 -0400 Committer: Ivan Bella Committed: Thu Aug 4 00:04:26 2016 -0400 ---------------------------------------------------------------------- .../bcfile/BoundedRangeFileInputStream.java | 31 ++++++++------------ 1 file changed, 12 insertions(+), 19 deletions(-) ---------------------------------------------------------------------- http://git-wip-us.apache.org/repos/asf/accumulo/blob/9038d8b3/core/src/main/java/org/apache/accumulo/core/file/rfile/bcfile/BoundedRangeFileInputStream.java ---------------------------------------------------------------------- diff --git a/core/src/main/java/org/apache/accumulo/core/file/rfile/bcfile/BoundedRangeFileInputStream.java b/core/src/main/java/org/apache/accumulo/core/file/rfile/bcfile/BoundedRangeFileInputStream.java index b182b26..876b585 100644 --- a/core/src/main/java/org/apache/accumulo/core/file/rfile/bcfile/BoundedRangeFileInputStream.java +++ b/core/src/main/java/org/apache/accumulo/core/file/rfile/bcfile/BoundedRangeFileInputStream.java @@ -31,6 +31,7 @@ import org.apache.hadoop.fs.FSDataInputStream; */ class BoundedRangeFileInputStream extends InputStream { + private volatile boolean closed = false; private FSDataInputStream in; private long pos; private long end; @@ -60,12 +61,6 @@ class BoundedRangeFileInputStream extends InputStream { this.mark = -1; } - private void check() throws IOException { - if (in == null) { - throw new IOException("Stream closed"); - } - } - @Override public int available() throws IOException { return (int) (end - pos); @@ -94,17 +89,18 @@ class BoundedRangeFileInputStream extends InputStream { if (n == 0) return -1; Integer ret = 0; - final FSDataInputStream inLocal = in; - check(); // ensuring inLocal is not null - synchronized (inLocal) { - check(); // ensuring in is not null in which case we were closed which would be followed by someone else reusing the decompressor - inLocal.seek(pos); + synchronized (in) { + // ensuring we are not closed which would be followed by someone else reusing the decompressor + if (closed) { + throw new IOException("Stream closed"); + } + in.seek(pos); try { ret = AccessController.doPrivileged(new PrivilegedExceptionAction() { @Override public Integer run() throws IOException { int ret = 0; - ret = inLocal.read(b, off, n); + ret = in.read(b, off, n); return ret; } }); @@ -149,13 +145,10 @@ class BoundedRangeFileInputStream extends InputStream { @Override public void close() { - final FSDataInputStream inLocal = in; - if (inLocal != null) { - // synchronize on the FSDataInputStream to ensure we block closing if in the read method - synchronized (inLocal) { - // Invalidate the state of the stream. - in = null; - } + // synchronize on the FSDataInputStream to ensure we are blocked if in the read method + synchronized (in) { + // Invalidate the state of the stream. + closed = true; } } }