You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by te...@apache.org on 2014/01/02 19:20:27 UTC

svn commit: r1554870 - in /hbase/trunk: hbase-common/src/main/java/org/apache/hadoop/hbase/io/LimitInputStream.java hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogReader.java

Author: tedyu
Date: Thu Jan  2 18:20:27 2014
New Revision: 1554870

URL: http://svn.apache.org/r1554870
Log:
HBASE-10255 Remove dependency on LimitInputStream - revert, discussion on-going


Added:
    hbase/trunk/hbase-common/src/main/java/org/apache/hadoop/hbase/io/LimitInputStream.java
Modified:
    hbase/trunk/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogReader.java

Added: hbase/trunk/hbase-common/src/main/java/org/apache/hadoop/hbase/io/LimitInputStream.java
URL: http://svn.apache.org/viewvc/hbase/trunk/hbase-common/src/main/java/org/apache/hadoop/hbase/io/LimitInputStream.java?rev=1554870&view=auto
==============================================================================
--- hbase/trunk/hbase-common/src/main/java/org/apache/hadoop/hbase/io/LimitInputStream.java (added)
+++ hbase/trunk/hbase-common/src/main/java/org/apache/hadoop/hbase/io/LimitInputStream.java Thu Jan  2 18:20:27 2014
@@ -0,0 +1,104 @@
+/*
+ *
+ * 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.io;
+
+import java.io.FilterInputStream;
+import java.io.IOException;
+import java.io.InputStream;
+
+import static com.google.common.base.Preconditions.checkArgument;
+import static com.google.common.base.Preconditions.checkNotNull;
+
+/**
+ * Copied from guava source code v15 (LimitedInputStream)
+ * Guava deprecated LimitInputStream in v14 and removed it in v15. Copying this class here
+ * allows to be compatible with guava 11 to 15+.
+ */
+public final class LimitInputStream extends FilterInputStream {
+  private long left;
+  private long mark = -1;
+
+  public LimitInputStream(InputStream in, long limit) {
+    super(in);
+    checkNotNull(in);
+    checkArgument(limit >= 0, "limit must be non-negative");
+    left = limit;
+  }
+
+  @Override
+  public int available() throws IOException {
+    return (int) Math.min(in.available(), left);
+  }
+
+  // it's okay to mark even if mark isn't supported, as reset won't work
+  @Override
+  public synchronized void mark(int readLimit) {
+    in.mark(readLimit);
+    mark = left;
+  }
+
+  @Override
+  public int read() throws IOException {
+    if (left == 0) {
+      return -1;
+    }
+
+    int result = in.read();
+    if (result != -1) {
+      --left;
+    }
+    return result;
+  }
+
+  @Override
+  public int read(byte[] b, int off, int len) throws IOException {
+    if (left == 0) {
+      return -1;
+    }
+
+    len = (int) Math.min(len, left);
+    int result = in.read(b, off, len);
+    if (result != -1) {
+      left -= result;
+    }
+    return result;
+  }
+
+  @Override
+  public synchronized void reset() throws IOException {
+    if (!in.markSupported()) {
+      throw new IOException("Mark not supported");
+    }
+    if (mark == -1) {
+      throw new IOException("Mark not set");
+    }
+
+    in.reset();
+    left = mark;
+  }
+
+  @Override
+  public long skip(long n) throws IOException {
+    n = Math.min(n, left);
+    long skipped = in.skip(n);
+    left -= skipped;
+    return skipped;
+  }
+}

Modified: hbase/trunk/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogReader.java
URL: http://svn.apache.org/viewvc/hbase/trunk/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogReader.java?rev=1554870&r1=1554869&r2=1554870&view=diff
==============================================================================
--- hbase/trunk/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogReader.java (original)
+++ hbase/trunk/hbase-server/src/main/java/org/apache/hadoop/hbase/regionserver/wal/ProtobufLogReader.java Thu Jan  2 18:20:27 2014
@@ -25,13 +25,13 @@ import java.io.InputStream;
 import java.nio.ByteBuffer;
 import java.util.Arrays;
 
-import org.apache.commons.io.input.BoundedInputStream;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FSDataInputStream;
 import org.apache.hadoop.hbase.codec.Codec;
+import org.apache.hadoop.hbase.io.LimitInputStream;
 import org.apache.hadoop.hbase.protobuf.generated.WALProtos;
 import org.apache.hadoop.hbase.protobuf.generated.WALProtos.WALHeader.Builder;
 import org.apache.hadoop.hbase.protobuf.generated.WALProtos.WALKey;
@@ -226,7 +226,7 @@ public class ProtobufLogReader extends R
                 "inputStream.available()= " + this.inputStream.available() + ", " +
                 "entry size= " + size);
           }
-          final InputStream limitedInput = new BoundedInputStream(this.inputStream, size);
+          final InputStream limitedInput = new LimitInputStream(this.inputStream, size);
           builder.mergeFrom(limitedInput);
         } catch (InvalidProtocolBufferException ipbe) {
           throw (EOFException) new EOFException("Invalid PB, EOF? Ignoring; originalPosition=" +