|
17 | 17 | package io.openmessaging.storage.dledger; |
18 | 18 |
|
19 | 19 | import com.alibaba.fastjson.JSON; |
| 20 | + |
| 21 | +import io.openmessaging.storage.dledger.DLedgerEntryPusher.EntryDispatcherState; |
20 | 22 | import io.openmessaging.storage.dledger.common.Closure; |
21 | 23 | import io.openmessaging.storage.dledger.common.ShutdownAbleThread; |
22 | 24 | import io.openmessaging.storage.dledger.common.Status; |
|
55 | 57 | import java.util.concurrent.ConcurrentHashMap; |
56 | 58 | import java.util.concurrent.ConcurrentMap; |
57 | 59 | import java.util.concurrent.TimeUnit; |
| 60 | +import java.util.concurrent.atomic.AtomicLong; |
58 | 61 | import java.util.concurrent.atomic.AtomicReference; |
59 | 62 | import java.util.concurrent.locks.ReentrantLock; |
60 | 63 | import java.util.stream.Collectors; |
@@ -180,12 +183,33 @@ public int getPendingCount(long currTerm) { |
180 | 183 | return pendings.size(); |
181 | 184 | } |
182 | 185 |
|
| 186 | + public long getPendingSize(long currTerm) { |
| 187 | + if (dispatcherMap == null) { |
| 188 | + return 0; |
| 189 | + } |
| 190 | + long total = 0; |
| 191 | + for (EntryDispatcher dispatcher : dispatcherMap.values()) { |
| 192 | + total += dispatcher.pendingTotalSize.get(); |
| 193 | + } |
| 194 | + return total; |
| 195 | + } |
| 196 | + |
183 | 197 | public boolean isPendingFull(long currTerm) { |
184 | 198 | checkTermForPendingMap(currTerm, "isPendingFull"); |
185 | 199 | if (dLedgerStore.isLocalToomuchUncommitted()) { |
186 | 200 | return true; |
187 | 201 | } |
188 | | - return pendingClosure.get(currTerm).size() > dLedgerConfig.getMaxPendingRequestsNum(); |
| 202 | + if (pendingClosure.get(currTerm).size() > dLedgerConfig.getMaxPendingRequestsNum()) { |
| 203 | + return true; |
| 204 | + } |
| 205 | + // avoid too much memory in pending if more than half followers fall behind too much |
| 206 | + int fallBehindTooMuch = 0; |
| 207 | + for (EntryDispatcher dispatcher : dispatcherMap.values()) { |
| 208 | + if (dispatcher.pendingTotalSize.get() >= dLedgerConfig.getPeerPushPendingMaxBytes()) { |
| 209 | + fallBehindTooMuch++; |
| 210 | + } |
| 211 | + } |
| 212 | + return fallBehindTooMuch > dispatcherMap.size() / 2; |
189 | 213 | } |
190 | 214 |
|
191 | 215 | public void appendClosure(Closure closure, long term, long index) { |
@@ -409,6 +433,8 @@ private class EntryDispatcher extends ShutdownAbleThread { |
409 | 433 | private long matchIndex = -1; |
410 | 434 |
|
411 | 435 | private final int maxPendingSize = 1000; |
| 436 | + private AtomicLong pendingTotalSize = new AtomicLong(0); |
| 437 | + |
412 | 438 | private long term = -1; |
413 | 439 | private String leaderId = null; |
414 | 440 | private long lastCheckLeakTimeMs = System.currentTimeMillis(); |
@@ -715,6 +741,10 @@ private void doAppend() throws Exception { |
715 | 741 | doCheckAppendResponse(); |
716 | 742 | break; |
717 | 743 | } |
| 744 | + if (pendingTotalSize.get() >= dLedgerConfig.getPeerPushPendingMaxBytes()) { |
| 745 | + // to avoid oom or fullgc, we should wait for a while if too much pending big entry size |
| 746 | + break; |
| 747 | + } |
718 | 748 | long lastIndexToBeSend = doAppendInner(writeIndex); |
719 | 749 | if (lastIndexToBeSend == -1) { |
720 | 750 | break; |
@@ -759,8 +789,10 @@ private void sendBatchAppendEntryRequest() throws Exception { |
759 | 789 | StopWatch watch = StopWatch.createStarted(); |
760 | 790 | CompletableFuture<PushEntryResponse> responseFuture = dLedgerRpcService.push(batchAppendEntryRequest); |
761 | 791 | pendingMap.put(firstIndex, new Pair<>(System.currentTimeMillis(), batchAppendEntryRequest.getCount())); |
| 792 | + pendingTotalSize.addAndGet(entriesSize); |
762 | 793 | responseFuture.whenComplete((x, ex) -> { |
763 | 794 | try { |
| 795 | + pendingTotalSize.addAndGet(-1 * entriesSize); |
764 | 796 | PreConditions.check(ex == null, DLedgerResponseCode.UNKNOWN); |
765 | 797 | DLedgerResponseCode responseCode = DLedgerResponseCode.valueOf(x.getCode()); |
766 | 798 | switch (responseCode) { |
|
0 commit comments