This is an automated email from the ASF dual-hosted git repository.

zhaijia pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/bookkeeper.git


The following commit(s) were added to refs/heads/master by this push:
     new 7108ab4  ISSUE #750: support ByteBuf, ByteBuffer, byte[] in both 
WriteHandle, WriteAdvHandle, ReadHandle
7108ab4 is described below

commit 7108ab4b8e8116dd48c24fdaf845a28e048d7836
Author: Sijie Guo <[email protected]>
AuthorDate: Thu Nov 23 16:07:31 2017 +0800

    ISSUE #750: support ByteBuf, ByteBuffer, byte[] in both WriteHandle, 
WriteAdvHandle, ReadHandle
    
    Descriptions of the changes in this PR:
    
    - WriteHandle supports append byte[], ByteBuf and ByteBuffer
    - WriteAdvHandle supports write byte[], ByteBuf and ByteBuffer
    - LedgerEntry returns byte[], ByteBuffer and ByteBuf
    
    Author: Sijie Guo <[email protected]>
    
    Reviewers: Ivan Kelly <[email protected]>, Enrico Olivelli 
<[email protected]>, Jia Zhai <None>
    
    This closes #755 from sijie/issue_750, closes #750
---
 .../apache/bookkeeper/client/api/LedgerEntry.java  |  28 ++++-
 .../bookkeeper/client/api/WriteAdvHandle.java      |  26 ++++-
 .../apache/bookkeeper/client/api/WriteHandle.java  |  29 ++++-
 .../bookkeeper/client/impl/LedgerEntryImpl.java    |  14 ++-
 .../apache/bookkeeper/client/TestParallelRead.java |   6 +-
 .../bookkeeper/client/api/BookKeeperApiTest.java   |   4 +-
 .../bookkeeper/client/api/WriteAdvHandleTest.java  | 105 ++++++++++++++++++
 .../bookkeeper/client/api/WriteHandleTest.java     | 102 ++++++++++++++++++
 .../client/impl/LedgerEntryImplTest.java           | 118 +++++++++++++++++++++
 9 files changed, 417 insertions(+), 15 deletions(-)

diff --git 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/LedgerEntry.java
 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/LedgerEntry.java
index 43bdc34..0f54961 100644
--- 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/LedgerEntry.java
+++ 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/LedgerEntry.java
@@ -21,11 +21,16 @@
 package org.apache.bookkeeper.client.api;
 
 import io.netty.buffer.ByteBuf;
+import java.nio.ByteBuffer;
 import org.apache.bookkeeper.common.annotation.InterfaceAudience.Public;
 import org.apache.bookkeeper.common.annotation.InterfaceStability.Unstable;
 
 /**
- * An entry.
+ * An entry in a ledger.
+ *
+ * <p>The entry implementation may hold references to byte buffers under the 
hood. The users holding the
+ * references to the instances of this class, are responsible for calling 
{@link LedgerEntry#close()} to release
+ * resources held by the entry instances.
  *
  * @since 4.6
  */
@@ -55,14 +60,27 @@ public interface LedgerEntry extends AutoCloseable {
     long getLength();
 
     /**
-     * Returns the content of the entry.
+     * Returns the content of the entry as a byte array.
      *
      * @return the content of the entry
      */
-    byte[] getEntry();
+    byte[] getEntryBytes();
 
     /**
-     * Return the internal buffer that contains the entry payload.
+     * Exposes this entry's data as an NIO {@link ByteBuffer}. The returned 
buffer
+     * shares the content with this underneath bytebuf (which you can get it 
by {@link #getEntryBuffer()}).
+     * Changing the position and limit of the returned NIO buffer does not 
affect the indexes and
+     * marks of this underneath buffer.  This method is identical
+     * to {@code entry.getEntryBuffer().nioBuffer()}. This method does not
+     * modify {@code readerIndex} or {@code writerIndex} of the underlying 
bytebuf.
+     */
+    ByteBuffer getEntryNioBuffer();
+
+    /**
+     * Return the internal {@link ByteBuf} that contains the entry payload.
+     *
+     * <p>This call doesn't change the reference count on the returned 
bytebuf. If you want to use the bytebuf
+     * after the entry is released (via {@link #close()}, the caller must 
retain the references of the bytebuf.
      *
      * @return a ByteBuf which contains the data
      */
@@ -73,7 +91,7 @@ public interface LedgerEntry extends AutoCloseable {
      *
      * <p>This call will retain a slice of the underneath byte buffer.
      *
-     * @return a duplicated ledger entry.
+     * @return a duplicated ledger entry
      */
     LedgerEntry duplicate();
 
diff --git 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteAdvHandle.java
 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteAdvHandle.java
index a324c48..a6fb252 100644
--- 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteAdvHandle.java
+++ 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteAdvHandle.java
@@ -45,7 +45,7 @@ public interface WriteAdvHandle extends ReadHandle {
      *
      * @param entryId entryId to be added
      * @param data array of bytes to be written
-     * @return an handle to the result, in case of success it will return the 
same value of param entryId
+     * @return an handle to the result, in case of success it will return the 
same value of param entryId.
      */
     default CompletableFuture<Long> write(final long entryId, final ByteBuffer 
data) {
         return write(entryId, Unpooled.wrappedBuffer(data));
@@ -54,6 +54,30 @@ public interface WriteAdvHandle extends ReadHandle {
     /**
      * Add entry asynchronously to an open ledger.
      *
+     * @param entryId entryId to be added.
+     * @param data array of bytes to be written
+     * @return an handle to the result, in case of success it will return the 
same value of param {@code entryId}.
+     */
+    default CompletableFuture<Long> write(final long entryId, final byte[] 
data) {
+        return write(entryId, Unpooled.wrappedBuffer(data));
+    }
+
+    /**
+     * Add entry asynchronously to an open ledger.
+     *
+     * @param entryId entryId to  be added.
+     * @param data array of bytes to be written
+     * @param offset the offset of the bytes array
+     * @param length the length to data to write
+     * @return an handle to the result, in case of success it will return the 
same value of param {@code entryId}.
+     */
+    default CompletableFuture<Long> write(final long entryId, final byte[] 
data, int offset, int length) {
+        return write(entryId, Unpooled.wrappedBuffer(data, offset, length));
+    }
+
+    /**
+     * Add entry asynchronously to an open ledger.
+     *
      * @param entryId entryId to be added
      * @param data array of bytes to be written
      * @return an handle to the result, in case of success it will return the 
same value of param entryId
diff --git 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteHandle.java
 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteHandle.java
index 794d0bc..cefd749 100644
--- 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteHandle.java
+++ 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/api/WriteHandle.java
@@ -41,7 +41,8 @@ public interface WriteHandle extends ReadHandle {
     /**
      * Add entry asynchronously to an open ledger.
      *
-     * @param data array of bytes to be written
+     * @param data a bytebuf to be written. The bytebuf's reference count will 
be decremented by 1 after the
+     *             completable future is returned
      * @return an handle to the result, in case of success it will return the 
id of the newly appended entry
      */
     CompletableFuture<Long> append(ByteBuf data);
@@ -57,10 +58,34 @@ public interface WriteHandle extends ReadHandle {
     }
 
     /**
+     * Add an entry asynchronously to an open ledger.
+     *
+     * @param data array of bytes to be written
+     * @return a completable future represents the add result, in case of 
success the future returns the entry id
+     *         of this newly appended entry
+     */
+    default CompletableFuture<Long> append(byte[] data) {
+        return append(Unpooled.wrappedBuffer(data));
+    }
+
+    /**
+     * Add an entry asynchronously to an open ledger.
+     *
+     * @param data array of bytes to be written
+     * @param offset the offset in the bytes array
+     * @param length the length of the bytes to be appended
+     * @return a completable future represents the add result, in case of 
success the future returns the entry id
+     *         of this newly appended entry
+     */
+    default CompletableFuture<Long> append(byte[] data, int offset, int 
length) {
+        return append(Unpooled.wrappedBuffer(data, offset, length));
+    }
+
+    /**
      * Get the entry id of the last entry that has been enqueued for addition 
(but
      * may not have possibly been persisted to the ledger).
      *
-     * @return the entry id of the last entry pushed or -1 if no entry has 
been pushed.
+     * @return the entry id of the last entry pushed or -1 if no entry has 
been pushed
      */
     long getLastAddPushed();
 
diff --git 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/impl/LedgerEntryImpl.java
 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/impl/LedgerEntryImpl.java
index b90f299..f7b36c1 100644
--- 
a/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/impl/LedgerEntryImpl.java
+++ 
b/bookkeeper-server/src/main/java/org/apache/bookkeeper/client/impl/LedgerEntryImpl.java
@@ -23,6 +23,7 @@ import io.netty.buffer.ByteBufUtil;
 import io.netty.util.Recycler;
 import io.netty.util.Recycler.Handle;
 import io.netty.util.ReferenceCountUtil;
+import java.nio.ByteBuffer;
 import org.apache.bookkeeper.client.api.LedgerEntry;
 
 /**
@@ -85,6 +86,7 @@ public class LedgerEntryImpl implements LedgerEntry {
     }
 
     public void setEntryBuf(ByteBuf buf) {
+        ReferenceCountUtil.release(entryBuf);
         this.entryBuf = buf;
     }
 
@@ -116,8 +118,8 @@ public class LedgerEntryImpl implements LedgerEntry {
      * {@inheritDoc}
      */
     @Override
-    public byte[] getEntry() {
-        return ByteBufUtil.getBytes(entryBuf);
+    public byte[] getEntryBytes() {
+        return ByteBufUtil.getBytes(entryBuf, entryBuf.readerIndex(), 
entryBuf.readableBytes(), false);
     }
 
     /**
@@ -132,6 +134,14 @@ public class LedgerEntryImpl implements LedgerEntry {
      * {@inheritDoc}
      */
     @Override
+    public ByteBuffer getEntryNioBuffer() {
+        return entryBuf.nioBuffer();
+    }
+
+    /**
+     * {@inheritDoc}
+     */
+    @Override
     public LedgerEntryImpl duplicate() {
         return duplicate(this);
     }
diff --git 
a/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/TestParallelRead.java
 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/TestParallelRead.java
index eff8c3a..0f3365e 100644
--- 
a/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/TestParallelRead.java
+++ 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/TestParallelRead.java
@@ -84,7 +84,7 @@ public class TestParallelRead extends 
BookKeeperClusterTestCase {
             assertTrue(entries.hasNext());
             LedgerEntry entry = entries.next();
             assertNotNull(entry);
-            assertEquals(i, Integer.parseInt(new String(entry.getEntry())));
+            assertEquals(i, Integer.parseInt(new 
String(entry.getEntryBytes())));
             entry.close();
             assertFalse(entries.hasNext());
         }
@@ -101,7 +101,7 @@ public class TestParallelRead extends 
BookKeeperClusterTestCase {
         while (iterator.hasNext()) {
             LedgerEntry entry = iterator.next();
             assertNotNull(entry);
-            assertEquals(numReads, Integer.parseInt(new 
String(entry.getEntry())));
+            assertEquals(numReads, Integer.parseInt(new 
String(entry.getEntryBytes())));
             entry.close();
             ++numReads;
         }
@@ -206,7 +206,7 @@ public class TestParallelRead extends 
BookKeeperClusterTestCase {
         while (entries.hasNext()) {
             LedgerEntry entry = entries.next();
             assertNotNull(entry);
-            assertEquals(numReads, Integer.parseInt(new 
String(entry.getEntry())));
+            assertEquals(numReads, Integer.parseInt(new 
String(entry.getEntryBytes())));
             ++numReads;
         }
         assertEquals(numEntries, numReads);
diff --git 
a/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/BookKeeperApiTest.java
 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/BookKeeperApiTest.java
index bbaa358..0350720 100644
--- 
a/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/BookKeeperApiTest.java
+++ 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/BookKeeperApiTest.java
@@ -208,7 +208,7 @@ public class BookKeeperApiTest extends 
MockBookKeeperTestCase {
             LastConfirmedAndEntry lastConfirmedAndEntry =
                 result(reader.readLastAddConfirmedAndEntry(0, 999, false));
             assertEquals(2L, lastConfirmedAndEntry.getLastAddConfirmed());
-            assertArrayEquals(data, 
lastConfirmedAndEntry.getEntry().getEntry());
+            assertArrayEquals(data, 
lastConfirmedAndEntry.getEntry().getEntryBytes());
             lastConfirmedAndEntry.close();
         }
     }
@@ -278,7 +278,7 @@ public class BookKeeperApiTest extends 
MockBookKeeperTestCase {
     private static void checkEntries(Iterable<LedgerEntry> entries, byte[] 
data)
         throws InterruptedException, BKException {
         for (LedgerEntry le : entries) {
-            assertArrayEquals(data, le.getEntry());
+            assertArrayEquals(data, le.getEntryBytes());
         }
     }
 
diff --git 
a/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/WriteAdvHandleTest.java
 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/WriteAdvHandleTest.java
new file mode 100644
index 0000000..24408d9
--- /dev/null
+++ 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/WriteAdvHandleTest.java
@@ -0,0 +1,105 @@
+/*
+ * 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.bookkeeper.client.api;
+
+import static com.google.common.base.Charsets.UTF_8;
+import static org.junit.Assert.assertArrayEquals;
+import static org.mockito.ArgumentMatchers.any;
+import static org.mockito.ArgumentMatchers.anyInt;
+import static org.mockito.ArgumentMatchers.anyLong;
+import static org.mockito.ArgumentMatchers.eq;
+import static org.mockito.Mockito.doAnswer;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.times;
+import static org.mockito.Mockito.verify;
+import static org.powermock.api.mockito.PowerMockito.when;
+
+import io.netty.buffer.ByteBuf;
+import io.netty.buffer.ByteBufUtil;
+import java.nio.ByteBuffer;
+import java.util.concurrent.LinkedBlockingQueue;
+import org.apache.bookkeeper.common.concurrent.FutureUtils;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TestName;
+
+/**
+ * Unit test for {@link WriteAdvHandle}.
+ */
+public class WriteAdvHandleTest {
+
+    @Rule
+    public final TestName runtime = new TestName();
+
+    private final long entryId;
+    private final WriteAdvHandle handle = mock(WriteAdvHandle.class);
+    private final LinkedBlockingQueue<ByteBuf> entryQueue;
+
+    public WriteAdvHandleTest() {
+        this.entryId = System.currentTimeMillis();
+        this.entryQueue = new LinkedBlockingQueue<>();
+        doAnswer(invocationOnMock -> {
+            ByteBuf buf = invocationOnMock.getArgument(1);
+            entryQueue.add(buf);
+            return FutureUtils.value(-1L);
+        }).when(handle).write(anyLong(), any(ByteBuf.class));
+        when(handle.write(anyLong(), any(byte[].class))).thenCallRealMethod();
+        when(handle.write(anyLong(), any(byte[].class), anyInt(), 
anyInt())).thenCallRealMethod();
+        when(handle.write(anyLong(), 
any(ByteBuffer.class))).thenCallRealMethod();
+    }
+
+    @Test
+    public void testAppendBytes() throws Exception {
+        byte[] testData = runtime.getMethodName().getBytes(UTF_8);
+        handle.write(entryId, testData);
+
+        ByteBuf buffer = entryQueue.take();
+        byte[] bufferData = ByteBufUtil.getBytes(buffer);
+        assertArrayEquals(testData, bufferData);
+        verify(handle, times(1)).write(eq(entryId), any(ByteBuf.class));
+    }
+
+    @Test
+    public void testAppendBytes2() throws Exception {
+        byte[] testData = runtime.getMethodName().getBytes(UTF_8);
+        handle.write(entryId, testData, 1, testData.length / 2);
+        byte[] expectedData = new byte[testData.length / 2];
+        System.arraycopy(testData, 1, expectedData, 0, testData.length / 2);
+
+        ByteBuf buffer = entryQueue.take();
+        byte[] bufferData = ByteBufUtil.getBytes(buffer);
+        assertArrayEquals(expectedData, bufferData);
+        verify(handle, times(1)).write(eq(entryId), any(ByteBuf.class));
+    }
+
+    @Test
+    public void testAppendByteBuffer() throws Exception {
+        byte[] testData = runtime.getMethodName().getBytes(UTF_8);
+        handle.write(entryId, ByteBuffer.wrap(testData, 1, testData.length / 
2));
+        byte[] expectedData = new byte[testData.length / 2];
+        System.arraycopy(testData, 1, expectedData, 0, testData.length / 2);
+
+        ByteBuf buffer = entryQueue.take();
+        byte[] bufferData = ByteBufUtil.getBytes(buffer);
+        assertArrayEquals(expectedData, bufferData);
+        verify(handle, times(1)).write(eq(entryId), any(ByteBuf.class));
+    }
+
+}
diff --git 
a/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/WriteHandleTest.java
 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/WriteHandleTest.java
new file mode 100644
index 0000000..fd7ac5e
--- /dev/null
+++ 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/api/WriteHandleTest.java
@@ -0,0 +1,102 @@
+/*
+ * 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.bookkeeper.client.api;
+
+import static com.google.common.base.Charsets.UTF_8;
+import static org.junit.Assert.assertArrayEquals;
+import static org.mockito.ArgumentMatchers.any;
+import static org.mockito.ArgumentMatchers.anyInt;
+import static org.mockito.Mockito.doAnswer;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.times;
+import static org.mockito.Mockito.verify;
+import static org.mockito.Mockito.when;
+
+import io.netty.buffer.ByteBuf;
+import io.netty.buffer.ByteBufUtil;
+import java.nio.ByteBuffer;
+import java.util.concurrent.LinkedBlockingQueue;
+import lombok.extern.slf4j.Slf4j;
+import org.apache.bookkeeper.common.concurrent.FutureUtils;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.TestName;
+
+/**
+ * Unit test for the default methods in {@link WriteHandle}.
+ */
+@Slf4j
+public class WriteHandleTest {
+
+    @Rule
+    public final TestName runtime = new TestName();
+
+    private final WriteHandle handle = mock(WriteHandle.class);
+    private final LinkedBlockingQueue<ByteBuf> entryQueue;
+
+    public WriteHandleTest() {
+        this.entryQueue = new LinkedBlockingQueue<>();
+        doAnswer(invocationOnMock -> {
+            ByteBuf buf = invocationOnMock.getArgument(0);
+            entryQueue.add(buf);
+            return FutureUtils.value(-1L);
+        }).when(handle).append(any(ByteBuf.class));
+        when(handle.append(any(byte[].class))).thenCallRealMethod();
+        when(handle.append(any(byte[].class), anyInt(), 
anyInt())).thenCallRealMethod();
+        when(handle.append(any(ByteBuffer.class))).thenCallRealMethod();
+    }
+
+    @Test
+    public void testAppendBytes() throws Exception {
+        byte[] testData = runtime.getMethodName().getBytes(UTF_8);
+        handle.append(testData);
+
+        ByteBuf buffer = entryQueue.take();
+        byte[] bufferData = ByteBufUtil.getBytes(buffer);
+        assertArrayEquals(testData, bufferData);
+        verify(handle, times(1)).append(any(ByteBuf.class));
+    }
+
+    @Test
+    public void testAppendBytes2() throws Exception {
+        byte[] testData = runtime.getMethodName().getBytes(UTF_8);
+        handle.append(testData, 1, testData.length / 2);
+        byte[] expectedData = new byte[testData.length / 2];
+        System.arraycopy(testData, 1, expectedData, 0, testData.length / 2);
+
+        ByteBuf buffer = entryQueue.take();
+        byte[] bufferData = ByteBufUtil.getBytes(buffer);
+        assertArrayEquals(expectedData, bufferData);
+        verify(handle, times(1)).append(any(ByteBuf.class));
+    }
+
+    @Test
+    public void testAppendByteBuffer() throws Exception {
+        byte[] testData = runtime.getMethodName().getBytes(UTF_8);
+        handle.append(ByteBuffer.wrap(testData, 1, testData.length / 2));
+        byte[] expectedData = new byte[testData.length / 2];
+        System.arraycopy(testData, 1, expectedData, 0, testData.length / 2);
+
+        ByteBuf buffer = entryQueue.take();
+        byte[] bufferData = ByteBufUtil.getBytes(buffer);
+        assertArrayEquals(expectedData, bufferData);
+        verify(handle, times(1)).append(any(ByteBuf.class));
+    }
+}
diff --git 
a/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/impl/LedgerEntryImplTest.java
 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/impl/LedgerEntryImplTest.java
new file mode 100644
index 0000000..ee71680
--- /dev/null
+++ 
b/bookkeeper-server/src/test/java/org/apache/bookkeeper/client/impl/LedgerEntryImplTest.java
@@ -0,0 +1,118 @@
+/*
+ * 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.bookkeeper.client.impl;
+
+import static com.google.common.base.Charsets.UTF_8;
+import static org.junit.Assert.assertArrayEquals;
+import static org.junit.Assert.assertEquals;
+
+import io.netty.buffer.ByteBuf;
+import io.netty.buffer.Unpooled;
+import java.nio.ByteBuffer;
+import org.junit.After;
+import org.junit.Test;
+
+/**
+ * Unit test for {@link LedgerEntryImpl}.
+ */
+public class LedgerEntryImplTest {
+
+    private final long ledgerId;
+    private final long entryId;
+    private final long length;
+    private final byte[] dataBytes;
+    private final ByteBuf dataBuf;
+    private final LedgerEntryImpl entryImpl;
+
+    public LedgerEntryImplTest() {
+        this.ledgerId = 1234L;
+        this.entryId = 3579L;
+        this.length = 200L;
+        this.dataBytes = "test-ledger-entry-impl".getBytes(UTF_8);
+        this.dataBuf = Unpooled.wrappedBuffer(dataBytes);
+        this.entryImpl = LedgerEntryImpl.create(
+            ledgerId,
+            entryId,
+            length,
+            dataBuf);
+    }
+
+    @After
+    public void teardown() {
+        this.entryImpl.close();
+        assertEquals(0, dataBuf.refCnt());
+    }
+
+    @Test
+    public void testGetters() {
+        assertEquals(ledgerId, entryImpl.getLedgerId());
+        assertEquals(entryId, entryImpl.getEntryId());
+        assertEquals(length, entryImpl.getLength());
+        assertArrayEquals(dataBytes, entryImpl.getEntryBytes());
+        // getEntry should not modify readerIndex
+        assertEquals(0, entryImpl.getEntryBuffer().readerIndex());
+        assertEquals(dataBytes.length, 
entryImpl.getEntryBuffer().readableBytes());
+        // getEntryNioBuffer should not modify readerIndex
+        ByteBuffer nioBuffer = entryImpl.getEntryNioBuffer();
+        assertEquals(dataBytes.length, nioBuffer.remaining());
+        byte[] readBytes = new byte[nioBuffer.remaining()];
+        nioBuffer.get(readBytes);
+        assertArrayEquals(dataBytes, readBytes);
+        assertEquals(0, entryImpl.getEntryBuffer().readerIndex());
+        assertEquals(dataBytes.length, 
entryImpl.getEntryBuffer().readableBytes());
+    }
+
+    @Test
+    public void testSetters() {
+        assertEquals(ledgerId, entryImpl.getLedgerId());
+        assertEquals(entryId, entryImpl.getEntryId());
+        assertEquals(length, entryImpl.getLength());
+
+        entryImpl.setLength(length * 2);
+        assertEquals(length * 2, entryImpl.getLength());
+
+        entryImpl.setEntryId(entryId * 2);
+        assertEquals(entryId * 2, entryImpl.getEntryId());
+
+        byte[] anotherBytes = "another-ledger-entry-impl".getBytes(UTF_8);
+        ByteBuf anotherBuf = Unpooled.wrappedBuffer(anotherBytes);
+
+        entryImpl.setEntryBuf(anotherBuf);
+        // set buf should release the original buf
+        assertEquals(0, dataBuf.refCnt());
+    }
+
+    @Test
+    public void testDuplicate() {
+        LedgerEntryImpl duplicatedEntry = LedgerEntryImpl.duplicate(entryImpl);
+
+        // the underneath buffer should have 2 entries referencing it
+        assertEquals(2, dataBuf.refCnt());
+
+        assertEquals(ledgerId, duplicatedEntry.getLedgerId());
+        assertEquals(entryId, duplicatedEntry.getEntryId());
+        assertEquals(length, duplicatedEntry.getLength());
+        assertArrayEquals(dataBytes, duplicatedEntry.getEntryBytes());
+
+        duplicatedEntry.close();
+        assertEquals(1, dataBuf.refCnt());
+    }
+
+}

-- 
To stop receiving notification emails like this one, please contact
['"[email protected]" <[email protected]>'].

Reply via email to