Class TimeWindows
The semantics of time-based aggregation windows are: Every T1 (advance) milliseconds, compute the aggregate total for T2 (size) milliseconds.
- If
advance < size
a hopping windows is defined:
it discretize a stream into overlapping windows, which implies that a record maybe contained in one and or more "adjacent" windows. - If
advance == size
a tumbling window is defined:
it discretize a stream into non-overlapping windows, which implies that a record is only ever contained in one and only one tumbling window.
TimeWindow
s 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
.
-
Field Summary
Modifier and TypeFieldDescriptionfinal long
The size of the window's advance interval in milliseconds, i.e., by how much a window moves forward relative to the previous one.final long
The size of the windows in milliseconds.Fields inherited from class org.apache.kafka.streams.kstream.Windows
DEPRECATED_DEFAULT_24_HR_GRACE_PERIOD, NO_GRACE_PERIOD
-
Method Summary
Modifier and TypeMethodDescriptionReturn 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.boolean
Deprecated.since 3.0.long
Return the window grace period (the time to admit out-of-order events after the end of the window.)int
hashCode()
static TimeWindows
Deprecated.since 3.0.static TimeWindows
ofSizeAndGrace
(Duration size, Duration afterWindowEnd) Return a window definition with the given window size, and with the advance interval being equal to the window size.static TimeWindows
ofSizeWithNoGrace
(Duration size) Return a window definition with the given window size, and with the advance interval being equal to the window size.long
size()
Return the size of the specified windows in milliseconds.toString()
windowsFor
(long timestamp) Create all windows that contain the provided timestamp, indexed by non-negative window start timestamps.
-
Field Details
-
sizeMs
public final long sizeMsThe size of the windows in milliseconds. -
advanceMs
public final long advanceMsThe size of the window's advance interval in milliseconds, i.e., by how much a window moves forward relative to the previous one.
-
-
Method Details
-
ofSizeWithNoGrace
Return a window definition with the given window size, and with the advance interval being equal to the window size. The time interval represented by the N-th window is:[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
.CAUTION: Using this method implicitly sets the grace period to zero, which means that any out-of-order records arriving after the window ends are considered late and will be dropped.
- Parameters:
size
- The size of the window- Returns:
- a new window definition with default no grace period. Note that this means out-of-order records arriving after the window end will be dropped
- Throws:
IllegalArgumentException
- if the specified window size is zero or negative or can't be represented aslong milliseconds
-
ofSizeAndGrace
public static TimeWindows ofSizeAndGrace(Duration size, Duration afterWindowEnd) throws IllegalArgumentException Return a window definition with the given window size, and with the advance interval being equal to the window size. The time interval represented by the N-th window is:[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
.Using this method explicitly sets the grace period to the duration specified by
afterWindowEnd
, which means that only out-of-order records arriving more than the grace period after the window end will be dropped. The window close, after which any incoming records are considered late and will be rejected, is defined aswindowEnd + afterWindowEnd
- Parameters:
size
- The size of the window. Must be larger than zeroafterWindowEnd
- The grace period to admit out-of-order events to a window. Must be non-negative.- Returns:
- a TimeWindows object with the specified size and the specified grace period
- Throws:
IllegalArgumentException
- ifafterWindowEnd
is negative or can't be represented aslong milliseconds
-
of
Deprecated.since 3.0. UseofSizeWithNoGrace(Duration)
} insteadReturn a window definition with the given window size, and with the advance interval being equal to the window size. The time interval represented by the N-th window is:[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
.- Parameters:
size
- The size of the window- Returns:
- a new window definition without specifying the grace period (default to 24 hours minus window
size
) - Throws:
IllegalArgumentException
- if the specified window size is zero or negative or can't be represented aslong milliseconds
-
advanceBy
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. The time interval represented by the N-th window is:[N * advance, N * advance + size)
.This provides the semantics of hopping windows, which are fixed-sized, overlapping windows.
- Parameters:
advance
- The advance interval ("hop") of the window, with the requirement that0 < advance.toMillis() <= sizeMs
.- Returns:
- a new window definition with default maintain duration of 1 day
- Throws:
IllegalArgumentException
- if the advance interval is negative, zero, or larger than the window size
-
windowsFor
Description copied from class:Windows
Create all windows that contain the provided timestamp, indexed by non-negative window start timestamps.- Specified by:
windowsFor
in classWindows<org.apache.kafka.streams.kstream.internals.TimeWindow>
- Parameters:
timestamp
- the timestamp window should get created for- Returns:
- a map of
windowStartTimestamp -> Window
entries
-
size
public long size()Description copied from class:Windows
Return the size of the specified windows in milliseconds. -
grace
Deprecated.since 3.0. UseofSizeAndGrace(Duration, Duration)
insteadReject out-of-order events that arrive more thanmillisAfterWindowEnd
after the end of its window.Delay is defined as (stream_time - record_timestamp).
- Parameters:
afterWindowEnd
- The grace period to admit out-of-order events to a window.- Returns:
- this updated builder
- Throws:
IllegalArgumentException
- ifafterWindowEnd
is negative or can't be represented aslong milliseconds
IllegalStateException
- ifgrace(Duration)
is called afterofSizeAndGrace(Duration, Duration)
orofSizeWithNoGrace(Duration)
-
gracePeriodMs
public long gracePeriodMs()Description copied from class:Windows
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).- Specified by:
gracePeriodMs
in classWindows<org.apache.kafka.streams.kstream.internals.TimeWindow>
-
equals
-
hashCode
public int hashCode() -
toString
-