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 JoinWindowsinstance defines a maximum time difference for ajoin over two streamson 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. JoinWindowsare 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 SummaryAll Methods Static Methods Instance Methods Concrete Methods Deprecated Methods Modifier and Type Method Description JoinWindowsafter(long timeDifferenceMs)Deprecated.Useafter(Duration)insteadJoinWindowsafter(Duration timeDifference)Changes the end window boundary totimeDifferencebut keep the start window boundary as is.JoinWindowsbefore(long timeDifferenceMs)Deprecated.Usebefore(Duration)instead.JoinWindowsbefore(Duration timeDifference)Changes the start window boundary totimeDifferencebut keep the end window boundary as is.booleanequals(Object o)JoinWindowsgrace(Duration afterWindowEnd)Reject out-of-order events that are delayed more thanafterWindowEndafter the end of its window.longgracePeriodMs()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).inthashCode()longmaintainMs()Deprecated.since 2.1.static JoinWindowsof(long timeDifferenceMs)Deprecated.Useof(Duration)instead.static JoinWindowsof(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 maxtimeDifferenceearlier or later than the timestamp of the record from the primary stream.longsize()Return the size of the specified windows in milliseconds.StringtoString()JoinWindowsuntil(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 maxtimeDifferenceMsearlier or later than the timestamp of the record from the primary stream.- Parameters:
- timeDifferenceMs- join window interval in milliseconds
- Throws:
- IllegalArgumentException- if- timeDifferenceMsis negative
 
 - 
ofpublic 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 maxtimeDifferenceearlier or later than the timestamp of the record from the primary stream.- Parameters:
- timeDifference- join window interval
- Throws:
- IllegalArgumentException- if- timeDifferenceis negative or can't be represented as- long milliseconds
 
 - 
before@Deprecated public JoinWindows before(long timeDifferenceMs) throws IllegalArgumentException Deprecated.Usebefore(Duration)instead.Changes the start window boundary totimeDifferenceMsbut 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 mosttimeDifferenceMsearlier than the timestamp of the record from the primary stream.timeDifferenceMscan 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
 
 - 
beforepublic JoinWindows before(Duration timeDifference) throws IllegalArgumentException Changes the start window boundary totimeDifferencebut 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 mosttimeDifferenceearlier than the timestamp of the record from the primary stream.timeDifferencecan 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 or- timeDifferencecan't be represented as- long milliseconds
 
 - 
after@Deprecated public JoinWindows after(long timeDifferenceMs) throws IllegalArgumentException Deprecated.Useafter(Duration)insteadChanges the end window boundary totimeDifferenceMsbut 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 mosttimeDifferenceMslater than the timestamp of the record from the primary stream.timeDifferenceMscan 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
 
 - 
afterpublic JoinWindows after(Duration timeDifference) throws IllegalArgumentException Changes the end window boundary totimeDifferencebut 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 mosttimeDifferencelater than the timestamp of the record from the primary stream.timeDifferencecan 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 or- timeDifferencecan't be represented as- long milliseconds
 
 - 
windowsForpublic Map<Long,Window> windowsFor(long timestamp) Not supported byJoinWindows. ThrowsUnsupportedOperationException.- Specified by:
- windowsForin class- Windows<Window>
- Parameters:
- timestamp- the timestamp window should get created for
- Returns:
- a map of windowStartTimestamp -> Windowentries
- Throws:
- UnsupportedOperationException- at every invocation
 
 - 
sizepublic long size() Description copied from class:WindowsReturn the size of the specified windows in milliseconds.
 - 
gracepublic JoinWindows grace(Duration afterWindowEnd) throws IllegalArgumentException Reject out-of-order events that are delayed more thanafterWindowEndafter 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 the- afterWindowEndis negative of can't be represented as- long milliseconds
 
 - 
gracePeriodMspublic long gracePeriodMs() Description copied from class:WindowsReturn 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:
- gracePeriodMsin class- Windows<Window>
 
 - 
until@Deprecated public JoinWindows until(long durationMs) throws IllegalArgumentException Deprecated.since 2.1. Usegrace(Duration)instead.Description copied from class:WindowsSet 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:
- untilin class- Windows<Window>
- Parameters:
- durationMs- the window retention time in milliseconds
- Returns:
- itself
- Throws:
- IllegalArgumentException- if- durationMsis 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 TimeWindowsthe maintain duration is at least as small as the window size.- Overrides:
- maintainMsin class- Windows<Window>
- Returns:
- the window maintain duration
 
 
- 
 
-