Package org.apache.kafka.streams.kstream
Class JoinWindows
- java.lang.Object
-
- org.apache.kafka.streams.kstream.Windows<Window>
-
- org.apache.kafka.streams.kstream.JoinWindows
-
public final class JoinWindows extends Windows<Window>
The window specifications used for joins.A
JoinWindows
instance defines a maximum time difference for ajoin over two streams
on the same key. In SQL-style you would express this join as
There are three different window configuration supported:SELECT * FROM stream1, stream2 WHERE stream1.key = stream2.key AND stream1.ts - before <= stream2.ts AND stream2.ts <= stream1.ts + after
- before = after = time-difference
- before = 0 and after = time-difference
- before = time-difference and after = 0
Both values (before and after) must not result in an "inverse" window, i.e., upper-interval bound cannot be smaller than lower-interval bound.
JoinWindows
are sliding windows, thus, they are aligned to the actual record timestamps. This implies, that each input record defines its own window with start and end time being relative to the record's timestamp.For time semantics, see
TimestampExtractor
.- See Also:
TimeWindows
,UnlimitedWindows
,SessionWindows
,KStream.join(KStream, ValueJoiner, JoinWindows)
,KStream.join(KStream, ValueJoiner, JoinWindows, StreamJoined)
,KStream.leftJoin(KStream, ValueJoiner, JoinWindows)
,KStream.leftJoin(KStream, ValueJoiner, JoinWindows, StreamJoined)
,KStream.outerJoin(KStream, ValueJoiner, JoinWindows)
,KStream.outerJoin(KStream, ValueJoiner, JoinWindows, StreamJoined)
,TimestampExtractor
-
-
Method Summary
All Methods Static Methods Instance Methods Concrete Methods Deprecated Methods Modifier and Type Method Description JoinWindows
after(long timeDifferenceMs)
Deprecated.Useafter(Duration)
insteadJoinWindows
after(Duration timeDifference)
Changes the end window boundary totimeDifference
but keep the start window boundary as is.JoinWindows
before(long timeDifferenceMs)
Deprecated.Usebefore(Duration)
instead.JoinWindows
before(Duration timeDifference)
Changes the start window boundary totimeDifference
but keep the end window boundary as is.boolean
equals(Object o)
JoinWindows
grace(Duration afterWindowEnd)
Reject out-of-order events that are delayed more thanafterWindowEnd
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.static JoinWindows
of(long timeDifferenceMs)
Deprecated.Useof(Duration)
instead.static JoinWindows
of(Duration timeDifference)
Specifies that records of the same key are joinable if their timestamps are withintimeDifference
, i.e., the timestamp of a record from the secondary stream is maxtimeDifference
earlier or later than the timestamp of the record from the primary stream.long
size()
Return the size of the specified windows in milliseconds.String
toString()
JoinWindows
until(long durationMs)
Deprecated.since 2.1.Map<Long,Window>
windowsFor(long timestamp)
Not supported byJoinWindows
.
-
-
-
Method Detail
-
of
@Deprecated public static JoinWindows of(long timeDifferenceMs) throws IllegalArgumentException
Deprecated.Useof(Duration)
instead.Specifies that records of the same key are joinable if their timestamps are withintimeDifferenceMs
, i.e., the timestamp of a record from the secondary stream is maxtimeDifferenceMs
earlier or later than the timestamp of the record from the primary stream.- Parameters:
timeDifferenceMs
- join window interval in milliseconds- Throws:
IllegalArgumentException
- iftimeDifferenceMs
is negative
-
of
public static JoinWindows of(Duration timeDifference) throws IllegalArgumentException
Specifies that records of the same key are joinable if their timestamps are withintimeDifference
, i.e., the timestamp of a record from the secondary stream is maxtimeDifference
earlier or later than the timestamp of the record from the primary stream.- Parameters:
timeDifference
- join window interval- Throws:
IllegalArgumentException
- iftimeDifference
is negative or can't be represented aslong milliseconds
-
before
@Deprecated public JoinWindows before(long timeDifferenceMs) throws IllegalArgumentException
Deprecated.Usebefore(Duration)
instead.Changes the start window boundary totimeDifferenceMs
but keep the end window boundary as is. Thus, records of the same key are joinable if the timestamp of a record from the secondary stream is at mosttimeDifferenceMs
earlier than the timestamp of the record from the primary stream.timeDifferenceMs
can be negative but its absolute value must not be larger than current window "after" value (which would result in a negative window size).- Parameters:
timeDifferenceMs
- relative window start time in milliseconds- Throws:
IllegalArgumentException
- if the resulting window size is negative
-
before
public JoinWindows before(Duration timeDifference) throws IllegalArgumentException
Changes the start window boundary totimeDifference
but keep the end window boundary as is. Thus, records of the same key are joinable if the timestamp of a record from the secondary stream is at mosttimeDifference
earlier than the timestamp of the record from the primary stream.timeDifference
can be negative but its absolute value must not be larger than current window "after" value (which would result in a negative window size).- Parameters:
timeDifference
- relative window start time- Throws:
IllegalArgumentException
- if the resulting window size is negative ortimeDifference
can't be represented aslong milliseconds
-
after
@Deprecated public JoinWindows after(long timeDifferenceMs) throws IllegalArgumentException
Deprecated.Useafter(Duration)
insteadChanges the end window boundary totimeDifferenceMs
but keep the start window boundary as is. Thus, records of the same key are joinable if the timestamp of a record from the secondary stream is at mosttimeDifferenceMs
later than the timestamp of the record from the primary stream.timeDifferenceMs
can be negative but its absolute value must not be larger than current window "before" value (which would result in a negative window size).- Parameters:
timeDifferenceMs
- relative window end time in milliseconds- Throws:
IllegalArgumentException
- if the resulting window size is negative
-
after
public JoinWindows after(Duration timeDifference) throws IllegalArgumentException
Changes the end window boundary totimeDifference
but keep the start window boundary as is. Thus, records of the same key are joinable if the timestamp of a record from the secondary stream is at mosttimeDifference
later than the timestamp of the record from the primary stream.timeDifference
can be negative but its absolute value must not be larger than current window "before" value (which would result in a negative window size).- Parameters:
timeDifference
- relative window end time- Throws:
IllegalArgumentException
- if the resulting window size is negative ortimeDifference
can't be represented aslong milliseconds
-
windowsFor
public Map<Long,Window> windowsFor(long timestamp)
Not supported byJoinWindows
. ThrowsUnsupportedOperationException
.- Specified by:
windowsFor
in classWindows<Window>
- Parameters:
timestamp
- the timestamp window should get created for- Returns:
- a map of
windowStartTimestamp -> Window
entries - Throws:
UnsupportedOperationException
- at every invocation
-
size
public long size()
Description copied from class:Windows
Return the size of the specified windows in milliseconds.
-
grace
public JoinWindows grace(Duration afterWindowEnd) throws IllegalArgumentException
Reject out-of-order events that are delayed more thanafterWindowEnd
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
- if theafterWindowEnd
is negative of can't be represented aslong milliseconds
-
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<Window>
-
until
@Deprecated public JoinWindows until(long durationMs) throws IllegalArgumentException
Deprecated.since 2.1. Usegrace(Duration)
instead.Description copied from class:Windows
Set the window maintain duration (retention time) in milliseconds. This retention time is a guaranteed lower bound for how long a window will be maintained.- Overrides:
until
in classWindows<Window>
- Parameters:
durationMs
- the window retention time in milliseconds- Returns:
- itself
- Throws:
IllegalArgumentException
- ifdurationMs
is smaller than the window size
-
maintainMs
@Deprecated public long maintainMs()
Deprecated.since 2.1. This function should not be used anymore, sinceuntil(long)
is deprecated in favor ofgrace(Duration)
.Return the window maintain duration (retention time) in milliseconds.For
TimeWindows
the maintain duration is at least as small as the window size.- Overrides:
maintainMs
in classWindows<Window>
- Returns:
- the window maintain duration
-
-