public final class TimeWindows extends Windows<org.apache.kafka.streams.kstream.internals.TimeWindow>
The semantics of time-based aggregation windows are: Every T1 (advance) milliseconds, compute the aggregate total for T2 (size) milliseconds.
advance < size a hopping windows is defined:advance == size a tumbling window is defined:TimeWindows are aligned to the epoch.
Aligned to the epoch means, that the first window starts at timestamp zero.
For example, hopping windows with size of 5000ms and advance of 3000ms, have window boundaries
[0;5000),[3000;8000),... and not [1000;6000),[4000;9000),... or even something "random" like [1452;6452),[4452;9452),...
For time semantics, see TimestampExtractor.
| Modifier and Type | Field and Description |
|---|---|
long |
advanceMs
The size of the window's advance interval in milliseconds, i.e., by how much a window moves forward relative to
the previous one.
|
long |
sizeMs
The size of the windows in milliseconds.
|
| Modifier and Type | Method and Description |
|---|---|
TimeWindows |
advanceBy(Duration advance)
Return a window definition with the original size, but advance ("hop") the window by the given interval, which
specifies by how much a window moves forward relative to the previous one.
|
TimeWindows |
advanceBy(long advanceMs)
Deprecated.
Use
advanceBy(Duration) instead |
boolean |
equals(Object o) |
TimeWindows |
grace(Duration afterWindowEnd)
Reject out-of-order events that arrive more than
millisAfterWindowEnd
after the end of its window. |
long |
gracePeriodMs()
Return the window grace period (the time to admit
out-of-order events after the end of the window.)
Delay is defined as (stream_time - record_timestamp).
|
int |
hashCode() |
long |
maintainMs()
Deprecated.
since 2.1. Use
Materialized.retention instead. |
static TimeWindows |
of(Duration size)
Return a window definition with the given window size, and with the advance interval being equal to the window
size.
|
static TimeWindows |
of(long sizeMs)
Deprecated.
Use
of(Duration) instead |
long |
size()
Return the size of the specified windows in milliseconds.
|
String |
toString() |
TimeWindows |
until(long durationMs)
Deprecated.
since 2.1. Use
Materialized.retention or directly configure the retention in a store supplier
and use Materialized.as(WindowBytesStoreSupplier). |
Map<Long,org.apache.kafka.streams.kstream.internals.TimeWindow> |
windowsFor(long timestamp)
Create all windows that contain the provided timestamp, indexed by non-negative window start timestamps.
|
public final long sizeMs
public final long advanceMs
@Deprecated public static TimeWindows of(long sizeMs) throws IllegalArgumentException
of(Duration) instead[N * size, N * size + size).
This provides the semantics of tumbling windows, which are fixed-sized, gap-less, non-overlapping windows.
Tumbling windows are a special case of hopping windows with advance == size.
sizeMs - The size of the window in millisecondsIllegalArgumentException - if the specified window size is zero or negativepublic static TimeWindows of(Duration size) throws IllegalArgumentException
[N * size, N * size + size).
This provides the semantics of tumbling windows, which are fixed-sized, gap-less, non-overlapping windows.
Tumbling windows are a special case of hopping windows with advance == size.
size - The size of the windowIllegalArgumentException - if the specified window size is zero or negative or can't be represented as long milliseconds@Deprecated public TimeWindows advanceBy(long advanceMs)
advanceBy(Duration) instead[N * advance, N * advance + size).
This provides the semantics of hopping windows, which are fixed-sized, overlapping windows.
advanceMs - The advance interval ("hop") in milliseconds of the window, with the requirement that 0 < advanceMs <= sizeMs.IllegalArgumentException - if the advance interval is negative, zero, or larger than the window sizepublic TimeWindows advanceBy(Duration advance)
[N * advance, N * advance + size).
This provides the semantics of hopping windows, which are fixed-sized, overlapping windows.
advance - The advance interval ("hop") of the window, with the requirement that 0 < advance.toMillis() <= sizeMs.IllegalArgumentException - if the advance interval is negative, zero, or larger than the window sizepublic Map<Long,org.apache.kafka.streams.kstream.internals.TimeWindow> windowsFor(long timestamp)
WindowswindowsFor in class Windows<org.apache.kafka.streams.kstream.internals.TimeWindow>timestamp - the timestamp window should get created forwindowStartTimestamp -> Window entriespublic long size()
Windowspublic TimeWindows grace(Duration afterWindowEnd) throws IllegalArgumentException
millisAfterWindowEnd
after the end of its window.
Delay is defined as (stream_time - record_timestamp).
afterWindowEnd - The grace period to admit out-of-order events to a window.IllegalArgumentException - if afterWindowEnd is negative or can't be represented as long millisecondspublic long gracePeriodMs()
WindowsgracePeriodMs in class Windows<org.apache.kafka.streams.kstream.internals.TimeWindow>@Deprecated public TimeWindows until(long durationMs) throws IllegalArgumentException
Materialized.retention or directly configure the retention in a store supplier
and use Materialized.as(WindowBytesStoreSupplier).Windowsuntil in class Windows<org.apache.kafka.streams.kstream.internals.TimeWindow>durationMs - the window retention timeIllegalArgumentException - if duration is smaller than the window size@Deprecated public long maintainMs()
Materialized.retention instead.
For TimeWindows the maintain duration is at least as small as the window size.
maintainMs in class Windows<org.apache.kafka.streams.kstream.internals.TimeWindow>