|
27 | 27 | import io.netty.buffer.ByteBuf; |
28 | 28 | import io.netty.buffer.ByteBufAllocator; |
29 | 29 | import io.netty.buffer.ByteBufUtil; |
| 30 | +import io.netty.buffer.PooledByteBufAllocator; |
30 | 31 | import io.netty.buffer.Unpooled; |
31 | 32 | import io.netty.buffer.UnpooledByteBufAllocator; |
| 33 | +import io.netty.channel.Channel; |
32 | 34 | import io.netty.channel.EventLoopGroup; |
33 | 35 | import io.netty.channel.nio.NioEventLoopGroup; |
34 | 36 | import io.netty.util.ReferenceCounted; |
35 | 37 | import io.netty.util.concurrent.DefaultThreadFactory; |
36 | 38 | import java.io.File; |
37 | 39 | import java.io.IOException; |
38 | 40 | import java.nio.ByteBuffer; |
| 41 | +import java.time.Duration; |
39 | 42 | import java.util.Arrays; |
40 | 43 | import java.util.concurrent.CountDownLatch; |
41 | 44 | import java.util.concurrent.Executors; |
42 | 45 | import java.util.concurrent.ScheduledExecutorService; |
| 46 | +import java.util.concurrent.atomic.AtomicBoolean; |
43 | 47 | import java.util.concurrent.atomic.AtomicInteger; |
44 | 48 | import java.util.concurrent.atomic.AtomicReference; |
| 49 | +import lombok.extern.slf4j.Slf4j; |
45 | 50 | import org.apache.bookkeeper.bookie.MockUncleanShutdownDetection; |
46 | 51 | import org.apache.bookkeeper.bookie.TestBookieImpl; |
47 | 52 | import org.apache.bookkeeper.client.BKException; |
48 | 53 | import org.apache.bookkeeper.client.BKException.Code; |
| 54 | +import org.apache.bookkeeper.client.BookKeeper; |
49 | 55 | import org.apache.bookkeeper.client.BookKeeperClientStats; |
50 | 56 | import org.apache.bookkeeper.client.BookieInfoReader.BookieInfo; |
51 | 57 | import org.apache.bookkeeper.client.api.WriteFlag; |
|
57 | 63 | import org.apache.bookkeeper.net.BookieSocketAddress; |
58 | 64 | import org.apache.bookkeeper.proto.BookieClient; |
59 | 65 | import org.apache.bookkeeper.proto.BookieClientImpl; |
| 66 | +import org.apache.bookkeeper.proto.BookieProtoEncoding; |
60 | 67 | import org.apache.bookkeeper.proto.BookieProtocol; |
61 | 68 | import org.apache.bookkeeper.proto.BookieServer; |
62 | 69 | import org.apache.bookkeeper.proto.BookkeeperInternalCallbacks.GetBookieInfoCallback; |
63 | 70 | import org.apache.bookkeeper.proto.BookkeeperInternalCallbacks.ReadEntryCallback; |
64 | 71 | import org.apache.bookkeeper.proto.BookkeeperInternalCallbacks.WriteCallback; |
65 | 72 | import org.apache.bookkeeper.proto.BookkeeperProtocol; |
66 | 73 | import org.apache.bookkeeper.proto.DataFormats; |
| 74 | +import org.apache.bookkeeper.proto.PerChannelBookieClient; |
| 75 | +import org.apache.bookkeeper.proto.PerChannelBookieClientPool; |
67 | 76 | import org.apache.bookkeeper.proto.checksum.DigestManager; |
68 | 77 | import org.apache.bookkeeper.stats.NullStatsLogger; |
69 | 78 | import org.apache.bookkeeper.test.TestStatsProvider.TestOpStatsLogger; |
70 | 79 | import org.apache.bookkeeper.test.TestStatsProvider.TestStatsLogger; |
71 | 80 | import org.apache.bookkeeper.util.ByteBufList; |
72 | 81 | import org.apache.bookkeeper.util.IOUtils; |
73 | 82 | import org.awaitility.Awaitility; |
| 83 | +import org.awaitility.reflect.WhiteboxImpl; |
74 | 84 | import org.junit.After; |
75 | 85 | import org.junit.Before; |
76 | 86 | import org.junit.Test; |
77 | 87 |
|
78 | 88 | /** |
79 | 89 | * Test the bookie client. |
80 | 90 | */ |
| 91 | +@Slf4j |
81 | 92 | public class BookieClientTest { |
82 | 93 | BookieServer bs; |
83 | 94 | File tmpDir; |
@@ -745,4 +756,164 @@ public void testBatchedReadWithMaxSizeLimitCase2() throws Exception { |
745 | 756 | assertTrue(Arrays.equals(kbData, bytes)); |
746 | 757 | } |
747 | 758 | } |
| 759 | + |
| 760 | + /** |
| 761 | + * Explain the stacks of "BookieClientImpl.addEntry" here |
| 762 | + * 1.`BookieClientImpl.addEntry`. |
| 763 | + * a.Retain the `ByteBuf` before get `PerChannelBookieClient`. We call this `ByteBuf` as `toSend` in the |
| 764 | + * following sections. `toSend.recCnf` is `2` now. |
| 765 | + * 2.`Get PerChannelBookieClient`. |
| 766 | + * 3.`ChannelReadyForAddEntryCallback.operationComplete` |
| 767 | + * a.`PerChannelBookieClient.addEntry` |
| 768 | + * a-1.Build a new ByteBuf for request command. We call this `ByteBuf` new as `request` in the following |
| 769 | + * sections. |
| 770 | + * a-2.`channle.writeAndFlush(request)` or release the ByteBuf when `channel` is switching. |
| 771 | + * Note the callback will be called immediately if the channel is switching. |
| 772 | + * b.Release the `ByteBuf` since it has been retained at `step 1`. `toSend.recCnf` should be `1` now. |
| 773 | + */ |
| 774 | + public void testDataRefCnfWhenReconnect(boolean useV2WireProtocol, boolean smallPayload, |
| 775 | + boolean withDelayReconnect, boolean withDelayAddEntry, |
| 776 | + int tryTimes) throws Exception { |
| 777 | + final long ledgerId = 1; |
| 778 | + final BookieId addr = bs.getBookieId(); |
| 779 | + // Build passwd. |
| 780 | + byte[] passwd = new byte[20]; |
| 781 | + Arrays.fill(passwd, (byte) 'a'); |
| 782 | + // Build digest manager. |
| 783 | + DigestManager digestManager = DigestManager.instantiate(1, passwd, |
| 784 | + BookKeeper.DigestType.toProtoDigestType(BookKeeper.DigestType.DUMMY), |
| 785 | + PooledByteBufAllocator.DEFAULT, useV2WireProtocol); |
| 786 | + // Build client. |
| 787 | + ClientConfiguration clientConf = new ClientConfiguration(); |
| 788 | + clientConf.setUseV2WireProtocol(useV2WireProtocol); |
| 789 | + BookieClientImpl client = new BookieClientImpl(clientConf, eventLoopGroup, |
| 790 | + UnpooledByteBufAllocator.DEFAULT, executor, scheduler, NullStatsLogger.INSTANCE, |
| 791 | + BookieSocketAddress.LEGACY_BOOKIEID_RESOLVER); |
| 792 | + |
| 793 | + // Inject a reconnect event. |
| 794 | + // 1. Get the channel that will be used. |
| 795 | + // 2. Call add entry. |
| 796 | + // 3. Another thread close the channel that is using. |
| 797 | + for (int i = 0; i < tryTimes; i++) { |
| 798 | + long entryId = i + 1; |
| 799 | + long lac = i; |
| 800 | + // Build payload. |
| 801 | + int payloadLen; |
| 802 | + ByteBuf payload; |
| 803 | + if (smallPayload) { |
| 804 | + payloadLen = 1; |
| 805 | + payload = PooledByteBufAllocator.DEFAULT.buffer(1); |
| 806 | + payload.writeByte(1); |
| 807 | + } else { |
| 808 | + payloadLen = BookieProtoEncoding.SMALL_ENTRY_SIZE_THRESHOLD; |
| 809 | + payload = PooledByteBufAllocator.DEFAULT.buffer(); |
| 810 | + byte[] bs = new byte[payloadLen]; |
| 811 | + payload.writeBytes(bs); |
| 812 | + } |
| 813 | + |
| 814 | + // Digest. |
| 815 | + ReferenceCounted bb = digestManager.computeDigestAndPackageForSending(entryId, lac, |
| 816 | + payloadLen * entryId, payload, passwd, BookieProtocol.FLAG_NONE); |
| 817 | + log.info("Before send. bb.refCnf: {}", bb.refCnt()); |
| 818 | + |
| 819 | + // Step: get the channel that will be used. |
| 820 | + PerChannelBookieClientPool perChannelBookieClientPool = client.lookupClient(addr); |
| 821 | + AtomicReference<PerChannelBookieClient> perChannelBookieClient = new AtomicReference<>(); |
| 822 | + perChannelBookieClientPool.obtain((rc, result) -> perChannelBookieClient.set(result), ledgerId); |
| 823 | + Awaitility.await().untilAsserted(() -> { |
| 824 | + assertNotNull(perChannelBookieClient.get()); |
| 825 | + }); |
| 826 | + |
| 827 | + // Step: Inject a reconnect event. |
| 828 | + final int delayMillis = i; |
| 829 | + new Thread(() -> { |
| 830 | + if (withDelayReconnect) { |
| 831 | + sleep(delayMillis); |
| 832 | + } |
| 833 | + Channel channel = WhiteboxImpl.getInternalState(perChannelBookieClient.get(), "channel"); |
| 834 | + if (channel != null) { |
| 835 | + channel.close(); |
| 836 | + } |
| 837 | + }).start(); |
| 838 | + if (withDelayAddEntry) { |
| 839 | + sleep(delayMillis); |
| 840 | + } |
| 841 | + |
| 842 | + // Step: add entry. |
| 843 | + AtomicBoolean callbackExecuted = new AtomicBoolean(); |
| 844 | + WriteCallback callback = (rc, lId, eId, socketAddr, ctx) -> { |
| 845 | + log.info("Writing is finished. rc: {}, withDelayReconnect: {}, withDelayAddEntry: {}, ledgerId: {}," |
| 846 | + + " entryId: {}, socketAddr: {}, ctx: {}", |
| 847 | + rc, withDelayReconnect, withDelayAddEntry, lId, eId, socketAddr, ctx); |
| 848 | + callbackExecuted.set(true); |
| 849 | + }; |
| 850 | + client.addEntry(addr, ledgerId, passwd, entryId, bb, callback, i, BookieProtocol.FLAG_NONE, false, |
| 851 | + WriteFlag.NONE); |
| 852 | + // Wait for adding entry is finish. |
| 853 | + Awaitility.await().untilAsserted(() -> assertTrue(callbackExecuted.get())); |
| 854 | + // The steps have be explained on the method description. |
| 855 | + // Since the step "3-a-2" always runs before the step "3-b", so the "callbackExecuted" will be finished |
| 856 | + // before the step "3-b". Add a sleep to wait the step "3-a-2" is finish. |
| 857 | + Thread.sleep(100); |
| 858 | + // Check the ref count. |
| 859 | + Awaitility.await().atMost(Duration.ofSeconds(60)).untilAsserted(() -> { |
| 860 | + assertEquals(1, bb.refCnt()); |
| 861 | + // V2 will release this original data if it is a small. |
| 862 | + if (!useV2WireProtocol && !smallPayload) { |
| 863 | + assertEquals(1, payload.refCnt()); |
| 864 | + } |
| 865 | + }); |
| 866 | + bb.release(); |
| 867 | + // V2 will release this original data if it is a small. |
| 868 | + if (!useV2WireProtocol && !smallPayload) { |
| 869 | + payload.release(); |
| 870 | + } |
| 871 | + } |
| 872 | + // cleanup. |
| 873 | + client.close(); |
| 874 | + } |
| 875 | + |
| 876 | + private void sleep(int milliSeconds) { |
| 877 | + try { |
| 878 | + if (milliSeconds > 0) { |
| 879 | + Thread.sleep(1); |
| 880 | + } |
| 881 | + } catch (InterruptedException e) { |
| 882 | + log.warn("Error occurs", e); |
| 883 | + } |
| 884 | + } |
| 885 | + |
| 886 | + /** |
| 887 | + * Relate to https://github.com/apache/bookkeeper/pull/4289. |
| 888 | + */ |
| 889 | + @Test |
| 890 | + public void testDataRefCnfWhenReconnectV2() throws Exception { |
| 891 | + // Large payload. |
| 892 | + // Run this test may not reproduce the issue, you can reproduce the issue this way: |
| 893 | + // 1. Add two break points. |
| 894 | + // a. At the line "Channel c = channel" in the method PerChannelBookieClient.addEntry. |
| 895 | + // b. At the line "channel = null" in the method "PerChannelBookieClient.channelInactive". |
| 896 | + // 2. Make the break point b to run earlier than the break point a during debugging. |
| 897 | + testDataRefCnfWhenReconnect(true, false, false, false, 10); |
| 898 | + testDataRefCnfWhenReconnect(true, false, true, false, 10); |
| 899 | + testDataRefCnfWhenReconnect(true, false, false, true, 10); |
| 900 | + |
| 901 | + // Small payload. |
| 902 | + // There is no issue without https://github.com/apache/bookkeeper/pull/4289, just add a test for this scenario. |
| 903 | + testDataRefCnfWhenReconnect(true, true, false, false, 10); |
| 904 | + testDataRefCnfWhenReconnect(true, true, true, false, 10); |
| 905 | + testDataRefCnfWhenReconnect(true, true, false, true, 10); |
| 906 | + } |
| 907 | + |
| 908 | + /** |
| 909 | + * Please see the comment of the scenario "Large payload" in the {@link #testDataRefCnfWhenReconnectV2()} if you |
| 910 | + * can not reproduce the issue when running this test. |
| 911 | + * Relate to https://github.com/apache/bookkeeper/pull/4289. |
| 912 | + */ |
| 913 | + @Test |
| 914 | + public void testDataRefCnfWhenReconnectV3() throws Exception { |
| 915 | + testDataRefCnfWhenReconnect(false, true, false, false, 10); |
| 916 | + testDataRefCnfWhenReconnect(false, true, true, false, 10); |
| 917 | + testDataRefCnfWhenReconnect(false, true, false, true, 10); |
| 918 | + } |
748 | 919 | } |
0 commit comments