@@ -87,85 +87,86 @@ object TimeWindowing extends Rule[LogicalPlan] {
87
87
88
88
val window = windowExpressions.head
89
89
90
+ // time window is provided as time column of window function, replace it with WindowTime
90
91
if (StructType .acceptsType(window.timeColumn.dataType)) {
91
- return p.transformExpressions {
92
+ p.transformExpressions {
92
93
case t : TimeWindow => t.copy(timeColumn = WindowTime (window.timeColumn))
93
94
}
94
- }
95
-
96
- val metadata = window.timeColumn match {
97
- case a : Attribute => a.metadata
98
- case _ => Metadata .empty
99
- }
100
-
101
- val newMetadata = new MetadataBuilder ()
102
- .withMetadata(metadata)
103
- .putBoolean(TimeWindow .marker, true )
104
- .build()
95
+ } else {
96
+ val metadata = window.timeColumn match {
97
+ case a : Attribute => a.metadata
98
+ case _ => Metadata .empty
99
+ }
105
100
106
- def getWindow (i : Int , dataType : DataType ): Expression = {
107
- val timestamp = PreciseTimestampConversion (window.timeColumn, dataType, LongType )
108
- val remainder = (timestamp - window.startTime) % window.slideDuration
109
- val lastStart = timestamp - CaseWhen (Seq ((LessThan (remainder, 0 ),
110
- remainder + window.slideDuration)), Some (remainder))
111
- val windowStart = lastStart - i * window.slideDuration
112
- val windowEnd = windowStart + window.windowDuration
101
+ val newMetadata = new MetadataBuilder ()
102
+ .withMetadata(metadata)
103
+ .putBoolean(TimeWindow .marker, true )
104
+ .build()
113
105
114
- // We make sure value fields are nullable since the dataType of TimeWindow defines them
115
- // as nullable.
116
- CreateNamedStruct (
117
- Literal (WINDOW_START ) ::
118
- PreciseTimestampConversion (windowStart, LongType , dataType).castNullable() ::
119
- Literal (WINDOW_END ) ::
120
- PreciseTimestampConversion (windowEnd, LongType , dataType).castNullable() ::
121
- Nil )
122
- }
106
+ def getWindow (i : Int , dataType : DataType ): Expression = {
107
+ val timestamp = PreciseTimestampConversion (window.timeColumn, dataType, LongType )
108
+ val remainder = (timestamp - window.startTime) % window.slideDuration
109
+ val lastStart = timestamp - CaseWhen (Seq ((LessThan (remainder, 0 ),
110
+ remainder + window.slideDuration)), Some (remainder))
111
+ val windowStart = lastStart - i * window.slideDuration
112
+ val windowEnd = windowStart + window.windowDuration
113
+
114
+ // We make sure value fields are nullable since the dataType of TimeWindow defines them
115
+ // as nullable.
116
+ CreateNamedStruct (
117
+ Literal (WINDOW_START ) ::
118
+ PreciseTimestampConversion (windowStart, LongType , dataType).castNullable() ::
119
+ Literal (WINDOW_END ) ::
120
+ PreciseTimestampConversion (windowEnd, LongType , dataType).castNullable() ::
121
+ Nil )
122
+ }
123
123
124
- val windowAttr = AttributeReference (
125
- WINDOW_COL_NAME , window.dataType, metadata = newMetadata)()
124
+ val windowAttr = AttributeReference (
125
+ WINDOW_COL_NAME , window.dataType, metadata = newMetadata)()
126
126
127
- if (window.windowDuration == window.slideDuration) {
128
- val windowStruct = Alias (getWindow(0 , window.timeColumn.dataType), WINDOW_COL_NAME )(
129
- exprId = windowAttr.exprId, explicitMetadata = Some (newMetadata))
127
+ if (window.windowDuration == window.slideDuration) {
128
+ val windowStruct = Alias (getWindow(0 , window.timeColumn.dataType), WINDOW_COL_NAME )(
129
+ exprId = windowAttr.exprId, explicitMetadata = Some (newMetadata))
130
130
131
- val replacedPlan = p transformExpressions {
132
- case t : TimeWindow => windowAttr
133
- }
131
+ val replacedPlan = p transformExpressions {
132
+ case t : TimeWindow => windowAttr
133
+ }
134
134
135
- // For backwards compatibility we add a filter to filter out nulls
136
- val filterExpr = IsNotNull (window.timeColumn)
135
+ // For backwards compatibility we add a filter to filter out nulls
136
+ val filterExpr = IsNotNull (window.timeColumn)
137
137
138
- replacedPlan.withNewChildren(
139
- Project (windowStruct +: child.output,
140
- Filter (filterExpr, child)) :: Nil )
141
- } else {
142
- val overlappingWindows =
143
- math.ceil(window.windowDuration * 1.0 / window.slideDuration).toInt
144
- val windows =
145
- Seq .tabulate(overlappingWindows)(i =>
146
- getWindow(i, window.timeColumn.dataType))
147
-
148
- val projections = windows.map(_ +: child.output)
149
-
150
- // When the condition windowDuration % slideDuration = 0 is fulfilled,
151
- // the estimation of the number of windows becomes exact one,
152
- // which means all produced windows are valid.
153
- val filterExpr =
154
- if (window.windowDuration % window.slideDuration == 0 ) {
155
- IsNotNull (window.timeColumn)
138
+ replacedPlan.withNewChildren(
139
+ Project (windowStruct +: child.output,
140
+ Filter (filterExpr, child)) :: Nil )
156
141
} else {
157
- window.timeColumn >= windowAttr.getField(WINDOW_START ) &&
158
- window.timeColumn < windowAttr.getField(WINDOW_END )
142
+ val overlappingWindows =
143
+ math.ceil(window.windowDuration * 1.0 / window.slideDuration).toInt
144
+ val windows =
145
+ Seq .tabulate(overlappingWindows)(i =>
146
+ getWindow(i, window.timeColumn.dataType))
147
+
148
+ val projections = windows.map(_ +: child.output)
149
+
150
+ // When the condition windowDuration % slideDuration = 0 is fulfilled,
151
+ // the estimation of the number of windows becomes exact one,
152
+ // which means all produced windows are valid.
153
+ val filterExpr =
154
+ if (window.windowDuration % window.slideDuration == 0 ) {
155
+ IsNotNull (window.timeColumn)
156
+ } else {
157
+ window.timeColumn >= windowAttr.getField(WINDOW_START ) &&
158
+ window.timeColumn < windowAttr.getField(WINDOW_END )
159
+ }
160
+
161
+ val substitutedPlan = Filter (filterExpr,
162
+ Expand (projections, windowAttr +: child.output, child))
163
+
164
+ val renamedPlan = p transformExpressions {
165
+ case t : TimeWindow => windowAttr
166
+ }
167
+
168
+ renamedPlan.withNewChildren(substitutedPlan :: Nil )
159
169
}
160
-
161
- val substitutedPlan = Filter (filterExpr,
162
- Expand (projections, windowAttr +: child.output, child))
163
-
164
- val renamedPlan = p transformExpressions {
165
- case t : TimeWindow => windowAttr
166
- }
167
-
168
- renamedPlan.withNewChildren(substitutedPlan :: Nil )
169
170
}
170
171
} else if (numWindowExpr > 1 ) {
171
172
throw QueryCompilationErrors .multiTimeWindowExpressionsNotSupportedError(p)
@@ -210,74 +211,74 @@ object SessionWindowing extends Rule[LogicalPlan] {
210
211
val session = sessionExpressions.head
211
212
212
213
if (StructType .acceptsType(session.timeColumn.dataType)) {
213
- return p transformExpressions {
214
+ p transformExpressions {
214
215
case t : SessionWindow => t.copy(timeColumn = WindowTime (session.timeColumn))
215
216
}
216
- }
217
+ } else {
218
+ val metadata = session.timeColumn match {
219
+ case a : Attribute => a.metadata
220
+ case _ => Metadata .empty
221
+ }
217
222
218
- val metadata = session.timeColumn match {
219
- case a : Attribute => a. metadata
220
- case _ => Metadata .empty
221
- }
223
+ val newMetadata = new MetadataBuilder ()
224
+ .withMetadata( metadata)
225
+ .putBoolean( SessionWindow .marker, true )
226
+ .build()
222
227
223
- val newMetadata = new MetadataBuilder ()
224
- .withMetadata(metadata)
225
- .putBoolean(SessionWindow .marker, true )
226
- .build()
227
-
228
- val sessionAttr = AttributeReference (
229
- SESSION_COL_NAME , session.dataType, metadata = newMetadata)()
230
-
231
- val sessionStart =
232
- PreciseTimestampConversion (session.timeColumn, session.timeColumn.dataType, LongType )
233
- val gapDuration = session.gapDuration match {
234
- case expr if expr.dataType == CalendarIntervalType =>
235
- expr
236
- case expr if Cast .canCast(expr.dataType, CalendarIntervalType ) =>
237
- Cast (expr, CalendarIntervalType )
238
- case other =>
239
- throw QueryCompilationErrors .sessionWindowGapDurationDataTypeError(other.dataType)
240
- }
241
- val sessionEnd = PreciseTimestampConversion (session.timeColumn + gapDuration,
242
- session.timeColumn.dataType, LongType )
243
-
244
- // We make sure value fields are nullable since the dataType of SessionWindow defines them
245
- // as nullable.
246
- val literalSessionStruct = CreateNamedStruct (
247
- Literal (SESSION_START ) ::
248
- PreciseTimestampConversion (sessionStart, LongType , session.timeColumn.dataType)
249
- .castNullable() ::
250
- Literal (SESSION_END ) ::
251
- PreciseTimestampConversion (sessionEnd, LongType , session.timeColumn.dataType)
252
- .castNullable() ::
253
- Nil )
254
-
255
- val sessionStruct = Alias (literalSessionStruct, SESSION_COL_NAME )(
256
- exprId = sessionAttr.exprId, explicitMetadata = Some (newMetadata))
228
+ val sessionAttr = AttributeReference (
229
+ SESSION_COL_NAME , session.dataType, metadata = newMetadata)()
230
+
231
+ val sessionStart =
232
+ PreciseTimestampConversion (session.timeColumn, session.timeColumn.dataType, LongType )
233
+ val gapDuration = session.gapDuration match {
234
+ case expr if expr.dataType == CalendarIntervalType =>
235
+ expr
236
+ case expr if Cast .canCast(expr.dataType, CalendarIntervalType ) =>
237
+ Cast (expr, CalendarIntervalType )
238
+ case other =>
239
+ throw QueryCompilationErrors .sessionWindowGapDurationDataTypeError(other.dataType)
240
+ }
241
+ val sessionEnd = PreciseTimestampConversion (session.timeColumn + gapDuration,
242
+ session.timeColumn.dataType, LongType )
257
243
258
- val replacedPlan = p transformExpressions {
259
- case s : SessionWindow => sessionAttr
260
- }
244
+ // We make sure value fields are nullable since the dataType of SessionWindow defines them
245
+ // as nullable.
246
+ val literalSessionStruct = CreateNamedStruct (
247
+ Literal (SESSION_START ) ::
248
+ PreciseTimestampConversion (sessionStart, LongType , session.timeColumn.dataType)
249
+ .castNullable() ::
250
+ Literal (SESSION_END ) ::
251
+ PreciseTimestampConversion (sessionEnd, LongType , session.timeColumn.dataType)
252
+ .castNullable() ::
253
+ Nil )
261
254
262
- val filterByTimeRange = if (gapDuration.foldable) {
263
- val interval = gapDuration.eval().asInstanceOf [CalendarInterval ]
264
- interval == null || interval.months + interval.days + interval.microseconds <= 0
265
- } else {
266
- true
267
- }
255
+ val sessionStruct = Alias (literalSessionStruct, SESSION_COL_NAME )(
256
+ exprId = sessionAttr.exprId, explicitMetadata = Some (newMetadata))
268
257
269
- // As same as tumbling window, we add a filter to filter out nulls.
270
- // And we also filter out events with negative or zero or invalid gap duration.
271
- val filterExpr = if (filterByTimeRange) {
272
- IsNotNull (session.timeColumn) &&
273
- (sessionAttr.getField(SESSION_END ) > sessionAttr.getField(SESSION_START ))
274
- } else {
275
- IsNotNull (session.timeColumn)
276
- }
258
+ val replacedPlan = p transformExpressions {
259
+ case s : SessionWindow => sessionAttr
260
+ }
277
261
278
- replacedPlan.withNewChildren(
279
- Filter (filterExpr,
280
- Project (sessionStruct +: child.output, child)) :: Nil )
262
+ val filterByTimeRange = if (gapDuration.foldable) {
263
+ val interval = gapDuration.eval().asInstanceOf [CalendarInterval ]
264
+ interval == null || interval.months + interval.days + interval.microseconds <= 0
265
+ } else {
266
+ true
267
+ }
268
+
269
+ // As same as tumbling window, we add a filter to filter out nulls.
270
+ // And we also filter out events with negative or zero or invalid gap duration.
271
+ val filterExpr = if (filterByTimeRange) {
272
+ IsNotNull (session.timeColumn) &&
273
+ (sessionAttr.getField(SESSION_END ) > sessionAttr.getField(SESSION_START ))
274
+ } else {
275
+ IsNotNull (session.timeColumn)
276
+ }
277
+
278
+ replacedPlan.withNewChildren(
279
+ Filter (filterExpr,
280
+ Project (sessionStruct +: child.output, child)) :: Nil )
281
+ }
281
282
} else if (numWindowExpr > 1 ) {
282
283
throw QueryCompilationErrors .multiTimeWindowExpressionsNotSupportedError(p)
283
284
} else {
0 commit comments