iamaleksey commented on code in PR #2982:
URL: https://github.com/apache/cassandra/pull/2982#discussion_r1453622777


##########
src/java/org/apache/cassandra/service/accord/AccordJournal.java:
##########
@@ -699,6 +1003,246 @@ private static int msVersion(int version)
         }
     }
 
+    /*
+     * Record framing logic
+     */
+
+    private final class FrameAggregator implements Interruptible.Task
+    {
+        /* external MPSC pending request queue */
+        private final ManyToOneConcurrentLinkedQueue<PendingRequest> 
unframedRequests = new ManyToOneConcurrentLinkedQueue<>();
+
+        private final LongArrayList waitForEpochs = new LongArrayList();
+        private final Long2ObjectHashMap<ArrayList<PendingRequest>> 
delayedRequests = new Long2ObjectHashMap<>();
+
+        private volatile Interruptible executor;
+
+        // a signal and flag that callers outside the aggregator thread can use
+        // to signal they want the aggregator to run again
+        private final Semaphore haveWork = newSemaphore(1);
+
+        void onWrite(Pointer pointer, RequestContext context)
+        {
+            unframedRequests.add(new PendingRequest(pointer, context));
+            haveWork.release(1);
+        }
+
+        void notifyOfEpoch()
+        {
+            haveWork.release(1);
+        }
+
+        void start()
+        {
+            executor = 
executorFactory().infiniteLoop("AccordJournal#FrameAggregator", this, SAFE, 
NON_DAEMON, SYNCHRONIZED);
+        }
+
+        void shutdown()
+        {
+            executor.shutdown();
+        }
+
+        /* internal reusable buffers used for frame generation */
+        private final ArrayList<PendingRequest> requestBuffer = new 
ArrayList<>();
+        private final ArrayList<Pointer> pointersBuffer = new ArrayList<>();
+        private final ArrayList<RequestContext> contextsBuffer = new 
ArrayList<>();
+
+        @Override
+        public void run(Interruptible.State state) throws InterruptedException
+        {
+            if (unframedRequests.isEmpty() && delayedRequests.isEmpty())
+                return;
+
+            try
+            {
+                doRun();
+            }
+            finally
+            {
+                requestBuffer.clear();
+                pointersBuffer.clear();
+                contextsBuffer.clear();
+            }
+
+            haveWork.acquire(1);
+        }
+
+        private void doRun()
+        {
+            /*
+             * Deal with delayed requests
+             */
+
+            waitForEpochs.sort(null);
+
+            for (int i = 0; i < waitForEpochs.size(); i++)
+            {
+                long waitForEpoch = waitForEpochs.getLong(i);
+                if (!node.topology().hasEpoch(waitForEpoch))
+                    break;
+                requestBuffer.addAll(delayedRequests.remove(waitForEpoch));
+            }
+
+            waitForEpochs.removeIfLong(epoch -> 
!delayedRequests.containsKey(epoch));
+
+            /*
+             * Deal with regular pending requests
+             */
+
+            PendingRequest request;
+            while (null != (request = unframedRequests.poll()))
+            {
+                long waitForEpoch = request.context.waitForEpoch;
+                if (!node.topology().hasEpoch(waitForEpoch))
+                {
+                    delayedRequests.computeIfAbsent(waitForEpoch, ignore -> 
new ArrayList<>()).add(request);
+                    if (!waitForEpochs.containsLong(waitForEpoch))
+                    {
+                        waitForEpochs.addLong(waitForEpoch);
+                        node.withEpoch(waitForEpoch, this::notifyOfEpoch);
+                    }
+                }
+                else
+                {
+                    requestBuffer.add(request);
+                }
+            }
+
+            for (PendingRequest req : requestBuffer)
+            {
+                pointersBuffer.add(req.pointer);
+                contextsBuffer.add(req.context);
+            }
+
+            if (!requestBuffer.isEmpty())
+            {
+                FrameRecord frame = new FrameRecord(node.uniqueNow(), 
pointersBuffer.toArray(new Pointer[0]));

Review Comment:
   I still needed to copy somewhere, as the original buffer was needed for 
later reuse. But I simplified the logic a little by switching to lists and not 
keeping additional buffer state between runs of aggregator. 
   
   P.S. As for escaping new empty arrays there, this is from 2016,  regarding 
`toArray()`: https://shipilev.net/blog/2016/arrays-wisdom-ancients/



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: [email protected]

For queries about this service, please contact Infrastructure at:
[email protected]


---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]

Reply via email to