1u0 commented on a change in pull request #9717: [FLINK-14044] [runtime] Reducing synchronization in AsyncWaitOperator URL: https://github.com/apache/flink/pull/9717#discussion_r329993151
########## File path: flink-streaming-java/src/main/java/org/apache/flink/streaming/api/operators/async/queue/UnorderedStreamElementQueue.java ########## @@ -19,287 +19,261 @@ package org.apache.flink.streaming.api.operators.async.queue; import org.apache.flink.annotation.Internal; -import org.apache.flink.streaming.api.operators.async.OperatorActions; +import org.apache.flink.streaming.api.functions.async.ResultFuture; +import org.apache.flink.streaming.api.operators.TimestampedCollector; +import org.apache.flink.streaming.api.watermark.Watermark; +import org.apache.flink.streaming.runtime.streamrecord.StreamElement; +import org.apache.flink.streaming.runtime.streamrecord.StreamRecord; import org.apache.flink.util.Preconditions; import org.slf4j.Logger; import org.slf4j.LoggerFactory; import java.util.ArrayDeque; -import java.util.Arrays; +import java.util.ArrayList; import java.util.Collection; +import java.util.Deque; import java.util.HashSet; -import java.util.Iterator; +import java.util.List; +import java.util.Optional; +import java.util.Queue; import java.util.Set; -import java.util.concurrent.Executor; -import java.util.concurrent.locks.Condition; -import java.util.concurrent.locks.ReentrantLock; /** - * Unordered implementation of the {@link StreamElementQueue}. The unordered stream element queue - * emits asynchronous results as soon as they are completed. Additionally it maintains the - * watermark-stream record order. This means that no stream record can be overtaken by a watermark - * and no watermark can overtake a stream record. However, stream records falling in the same - * segment between two watermarks can overtake each other (their emission order is not guaranteed). + * Unordered implementation of the {@link StreamElementQueue}. The unordered stream element queue provides + * asynchronous results as soon as they are completed. Additionally, it maintains the watermark-stream record order. + * + * <p>Elements can be logically grouped into different segments separated by watermarks. A segment needs to be + * completely emitted before entries from a following segment are emitted. Thus, no stream record can be overtaken + * by a watermark and no watermark can overtake a stream record. + * However, stream records falling in the same segment between two watermarks can overtake each other (their emission + * order is not guaranteed). */ @Internal -public class UnorderedStreamElementQueue implements StreamElementQueue { +public final class UnorderedStreamElementQueue<OUT> implements StreamElementQueue<OUT> { private static final Logger LOG = LoggerFactory.getLogger(UnorderedStreamElementQueue.class); /** Capacity of this queue. */ private final int capacity; - /** Executor to run the onComplete callbacks. */ - private final Executor executor; - - /** OperatorActions to signal the owning operator a failure. */ - private final OperatorActions operatorActions; - - /** Queue of uncompleted stream element queue entries segmented by watermarks. */ - private final ArrayDeque<Set<StreamElementQueueEntry<?>>> uncompletedQueue; - - /** Queue of completed stream element queue entries. */ - private final ArrayDeque<StreamElementQueueEntry<?>> completedQueue; - - /** First (chronologically oldest) uncompleted set of stream element queue entries. */ - private Set<StreamElementQueueEntry<?>> firstSet; + /** Queue of queue entries segmented by watermarks. */ + private final Deque<Segment<OUT>> segments; - // Last (chronologically youngest) uncompleted set of stream element queue entries. New - // stream element queue entries are inserted into this set. - private Set<StreamElementQueueEntry<?>> lastSet; - private volatile int numberEntries; - - /** Locks and conditions for the blocking queue. */ - private final ReentrantLock lock; - private final Condition notFull; - private final Condition hasCompletedEntries; - - public UnorderedStreamElementQueue( - int capacity, - Executor executor, - OperatorActions operatorActions) { + private int numberOfEntries; + public UnorderedStreamElementQueue(int capacity) { Preconditions.checkArgument(capacity > 0, "The capacity must be larger than 0."); - this.capacity = capacity; - - this.executor = Preconditions.checkNotNull(executor, "executor"); - this.operatorActions = Preconditions.checkNotNull(operatorActions, "operatorActions"); - - this.uncompletedQueue = new ArrayDeque<>(capacity); - this.completedQueue = new ArrayDeque<>(capacity); - - this.firstSet = new HashSet<>(capacity); - this.lastSet = firstSet; - - this.numberEntries = 0; - - this.lock = new ReentrantLock(); - this.notFull = lock.newCondition(); - this.hasCompletedEntries = lock.newCondition(); + this.capacity = capacity; + // most likely scenario are 4 segments <elements, watermark, elements, watermark> + this.segments = new ArrayDeque<>(4); + this.numberOfEntries = 0; } @Override - public <T> void put(StreamElementQueueEntry<T> streamElementQueueEntry) throws InterruptedException { - lock.lockInterruptibly(); - - try { - while (numberEntries >= capacity) { - notFull.await(); + public Optional<ResultFuture<OUT>> tryPut(StreamElement streamElement) { + if (size() < capacity) { + StreamElementQueueEntry<OUT> queueEntry; + if (streamElement.isRecord()) { + queueEntry = addRecord((StreamRecord<?>) streamElement); + } else if (streamElement.isWatermark()) { + queueEntry = addWatermark((Watermark) streamElement); + } else { + throw new UnsupportedOperationException("Cannot enqueue " + streamElement); } - addEntry(streamElementQueueEntry); - } finally { - lock.unlock(); - } - } - - @Override - public <T> boolean tryPut(StreamElementQueueEntry<T> streamElementQueueEntry) throws InterruptedException { - lock.lockInterruptibly(); - - try { - if (numberEntries < capacity) { - addEntry(streamElementQueueEntry); + numberOfEntries++; - LOG.debug("Put element into unordered stream element queue. New filling degree " + - "({}/{}).", numberEntries, capacity); + LOG.debug("Put element into unordered stream element queue. New filling degree " + + "({}/{}).", size(), capacity); - return true; - } else { - LOG.debug("Failed to put element into unordered stream element queue because it " + - "was full ({}/{}).", numberEntries, capacity); + return Optional.of(queueEntry); + } else { + LOG.debug("Failed to put element into unordered stream element queue because it " + + "was full ({}/{}).", size(), capacity); - return false; - } - } finally { - lock.unlock(); + return Optional.empty(); } } - @Override - public AsyncResult peekBlockingly() throws InterruptedException { - lock.lockInterruptibly(); + private StreamElementQueueEntry<OUT> addRecord(StreamRecord<?> record) { + // ensure that there is at least one segment + Segment<OUT> lastSegment; + if (segments.isEmpty()) { + lastSegment = addSegment(capacity); + } else { + lastSegment = segments.getLast(); + } - try { - while (completedQueue.isEmpty()) { - hasCompletedEntries.await(); - } + // entry is bound to segment to notify it easily upon completion + StreamElementQueueEntry<OUT> queueEntry = new SegmentdStreamRecordQueueEntry<>(record, lastSegment); + lastSegment.add(queueEntry); + return queueEntry; + } - LOG.debug("Peeked head element from unordered stream element queue with filling degree " + - "({}/{}).", numberEntries, capacity); + private Segment<OUT> addSegment(int capacity) { + Segment newSegment = new Segment(capacity); + segments.addLast(newSegment); + return newSegment; + } - return completedQueue.peek(); - } finally { - lock.unlock(); + private StreamElementQueueEntry<OUT> addWatermark(Watermark watermark) { + Segment<OUT> watermarkSegment; + if (!segments.isEmpty() && segments.getLast().isEmpty()) { + // reuse already existing segment if possible (completely drained) or the new segment added at the end of + // this method for two succeeding watermarks + watermarkSegment = segments.getLast(); + } else { + watermarkSegment = addSegment(1); } - } - @Override - public AsyncResult poll() throws InterruptedException { - lock.lockInterruptibly(); + StreamElementQueueEntry<OUT> watermarkEntry = new WatermarkQueueEntry<>(watermark); + watermarkSegment.add(watermarkEntry); - try { - while (completedQueue.isEmpty()) { - hasCompletedEntries.await(); - } + // add a new segment for actual elements + addSegment(capacity); + return watermarkEntry; + } - numberEntries--; - notFull.signalAll(); + @Override + public boolean hasCompletedElements() { + return !this.segments.isEmpty() && this.segments.getFirst().hasCompleted(); + } - LOG.debug("Polled element from unordered stream element queue. New filling degree " + - "({}/{}).", numberEntries, capacity); + @Override + public void emitCompletedElement(TimestampedCollector<OUT> output) { + if (segments.isEmpty()) { + return; + } + final Segment currentSegment = segments.getFirst(); + numberOfEntries -= currentSegment.emitCompleted(output); - return completedQueue.poll(); - } finally { - lock.unlock(); + // remove any segment if there are further segments, if not leave it as an optimization even if empty + if (segments.size() > 1 && currentSegment.isEmpty()) { + segments.pop(); } } @Override - public Collection<StreamElementQueueEntry<?>> values() throws InterruptedException { - lock.lockInterruptibly(); - - try { - StreamElementQueueEntry<?>[] array = new StreamElementQueueEntry[numberEntries]; - - array = completedQueue.toArray(array); - - int counter = completedQueue.size(); - - for (StreamElementQueueEntry<?> entry: firstSet) { - array[counter] = entry; - counter++; - } - - for (Set<StreamElementQueueEntry<?>> asyncBufferEntries : uncompletedQueue) { - - for (StreamElementQueueEntry<?> streamElementQueueEntry : asyncBufferEntries) { - array[counter] = streamElementQueueEntry; - counter++; - } - } - - return Arrays.asList(array); - } finally { - lock.unlock(); + public List<StreamElement> values() { + List<StreamElement> list = new ArrayList<>(); + for (Segment s : segments) { + s.addPendingElements(list); } + return list; } @Override public boolean isEmpty() { - return numberEntries == 0; + return numberOfEntries == 0; } @Override public int size() { - return numberEntries; + return numberOfEntries; } /** - * Callback for onComplete events for the given stream element queue entry. Whenever a queue - * entry is completed, it is checked whether this entry belongs to the first set. If this is the - * case, then the element is added to the completed entries queue from where it can be consumed. - * If the first set becomes empty, then the next set is polled from the uncompleted entries - * queue. Completed entries from this new set are then added to the completed entries queue. - * - * @param streamElementQueueEntry which has been completed - * @throws InterruptedException if the current thread has been interrupted while performing the - * on complete callback. + * An entry that notifies the respective segment upon completion. */ - public void onCompleteHandler(StreamElementQueueEntry<?> streamElementQueueEntry) throws InterruptedException { - lock.lockInterruptibly(); + static class SegmentdStreamRecordQueueEntry<OUT> extends StreamRecordQueueEntry<OUT> { Review comment: Typo `SegmentdStreamRecordQueueEntry` -> `{Segment|Segmented}StreamRecordQueueEntry`? ---------------------------------------------------------------- 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. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services