Class KeyGroupPartitionedPriorityQueue<T,PQ extends InternalPriorityQueue<T> & HeapPriorityQueueElement>
- java.lang.Object
-
- org.apache.flink.runtime.state.heap.KeyGroupPartitionedPriorityQueue<T,PQ>
-
- Type Parameters:
T
- the type of elements in the queue.PQ
- the type of sub-queue used for each key-group partition.
- All Implemented Interfaces:
InternalPriorityQueue<T>
,KeyGroupedInternalPriorityQueue<T>
public class KeyGroupPartitionedPriorityQueue<T,PQ extends InternalPriorityQueue<T> & HeapPriorityQueueElement> extends Object implements InternalPriorityQueue<T>, KeyGroupedInternalPriorityQueue<T>
This implementation ofInternalPriorityQueue
is internally partitioned into sub-queues per key-group and essentially works as a heap-of-heaps. Instances will have set semantics for elements if the sub-queues have set semantics.
-
-
Nested Class Summary
Nested Classes Modifier and Type Class Description static interface
KeyGroupPartitionedPriorityQueue.PartitionQueueSetFactory<T,PQS extends InternalPriorityQueue<T> & HeapPriorityQueueElement>
Factory that produces the sub-queues that represent the partitions of aKeyGroupPartitionedPriorityQueue
.
-
Constructor Summary
Constructors Constructor Description KeyGroupPartitionedPriorityQueue(KeyExtractorFunction<T> keyExtractor, PriorityComparator<T> elementPriorityComparator, KeyGroupPartitionedPriorityQueue.PartitionQueueSetFactory<T,PQ> orderedCacheFactory, KeyGroupRange keyGroupRange, int totalKeyGroups)
-
Method Summary
All Methods Instance Methods Concrete Methods Modifier and Type Method Description boolean
add(T toAdd)
Adds the given element to the set, if it is not already contained.void
addAll(Collection<? extends T> toAdd)
Adds all the given elements to the set.Set<T>
getSubsetForKeyGroup(int keyGroupId)
Returns the subset of elements in the priority queue that belongs to the given key-group, within the operator's key-group range.boolean
isEmpty()
Check if the set contains any elements.CloseableIterator<T>
iterator()
Iterator over all elements, no order guaranteed.T
peek()
Retrieves, but does not remove, the element (w.r.t. order) of this set, or returnsnull
if this set is empty.T
poll()
Retrieves and removes the first element (w.r.t. the order) of this set, or returnsnull
if this set is empty.boolean
remove(T toRemove)
Removes the given element from the set, if is contained in the set.int
size()
Returns the number of elements in this set.
-
-
-
Constructor Detail
-
KeyGroupPartitionedPriorityQueue
public KeyGroupPartitionedPriorityQueue(@Nonnull KeyExtractorFunction<T> keyExtractor, @Nonnull PriorityComparator<T> elementPriorityComparator, @Nonnull KeyGroupPartitionedPriorityQueue.PartitionQueueSetFactory<T,PQ> orderedCacheFactory, @Nonnull KeyGroupRange keyGroupRange, @Nonnegative int totalKeyGroups)
-
-
Method Detail
-
poll
@Nullable public T poll()
Description copied from interface:InternalPriorityQueue
Retrieves and removes the first element (w.r.t. the order) of this set, or returnsnull
if this set is empty.NOTE: Correct key (i.e. the key of the polled element) must be set on KeyContext before calling this method.
- Specified by:
poll
in interfaceInternalPriorityQueue<T>
- Returns:
- the first element of this ordered set, or
null
if this set is empty.
-
peek
@Nullable public T peek()
Description copied from interface:InternalPriorityQueue
Retrieves, but does not remove, the element (w.r.t. order) of this set, or returnsnull
if this set is empty.- Specified by:
peek
in interfaceInternalPriorityQueue<T>
- Returns:
- the first element (w.r.t. order) of this ordered set, or
null
if this set is empty.
-
add
public boolean add(@Nonnull T toAdd)
Description copied from interface:InternalPriorityQueue
Adds the given element to the set, if it is not already contained.- Specified by:
add
in interfaceInternalPriorityQueue<T>
- Parameters:
toAdd
- the element to add to the set.- Returns:
true
if the operation changed the head element or if it is unclear if the head element changed. Only returnsfalse
if the head element was not changed by this operation.
-
remove
public boolean remove(@Nonnull T toRemove)
Description copied from interface:InternalPriorityQueue
Removes the given element from the set, if is contained in the set.NOTE: Correct key (i.e. the key of the polled element) must be set on KeyContext before calling this method.
- Specified by:
remove
in interfaceInternalPriorityQueue<T>
- Parameters:
toRemove
- the element to remove.- Returns:
true
if the operation changed the head element or if it is unclear if the head element changed. Only returnsfalse
if the head element was not changed by this operation.
-
isEmpty
public boolean isEmpty()
Description copied from interface:InternalPriorityQueue
Check if the set contains any elements.- Specified by:
isEmpty
in interfaceInternalPriorityQueue<T>
- Returns:
- true if the set is empty, i.e. no element is contained.
-
size
public int size()
Description copied from interface:InternalPriorityQueue
Returns the number of elements in this set.- Specified by:
size
in interfaceInternalPriorityQueue<T>
- Returns:
- the number of elements in this set.
-
addAll
public void addAll(@Nullable Collection<? extends T> toAdd)
Description copied from interface:InternalPriorityQueue
Adds all the given elements to the set.- Specified by:
addAll
in interfaceInternalPriorityQueue<T>
-
iterator
@Nonnull public CloseableIterator<T> iterator()
Description copied from interface:InternalPriorityQueue
Iterator over all elements, no order guaranteed. Iterator must be closed after usage.- Specified by:
iterator
in interfaceInternalPriorityQueue<T>
-
getSubsetForKeyGroup
@Nonnull public Set<T> getSubsetForKeyGroup(int keyGroupId)
Description copied from interface:KeyGroupedInternalPriorityQueue
Returns the subset of elements in the priority queue that belongs to the given key-group, within the operator's key-group range.- Specified by:
getSubsetForKeyGroup
in interfaceKeyGroupedInternalPriorityQueue<T>
-
-