Skip to content

Commit f11aa7b

Browse files
mn-mikkemn-mikke
authored andcommitted
[SPARK-23821][SQL] Merging current master to the feature branch.
2 parents 9081291 + d5bec48 commit f11aa7b

File tree

11 files changed

+208
-37
lines changed

11 files changed

+208
-37
lines changed

python/pyspark/sql/functions.py

Lines changed: 17 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -1845,6 +1845,23 @@ def array_contains(col, value):
18451845
return Column(sc._jvm.functions.array_contains(_to_java_column(col), value))
18461846

18471847

1848+
@since(2.4)
1849+
def array_position(col, value):
1850+
"""
1851+
Collection function: Locates the position of the first occurrence of the given value
1852+
in the given array. Returns null if either of the arguments are null.
1853+
1854+
.. note:: The position is not zero based, but 1 based index. Returns 0 if the given
1855+
value could not be found in the array.
1856+
1857+
>>> df = spark.createDataFrame([(["c", "b", "a"],), ([],)], ['data'])
1858+
>>> df.select(array_position(df.data, "a")).collect()
1859+
[Row(array_position(data, a)=3), Row(array_position(data, a)=0)]
1860+
"""
1861+
sc = SparkContext._active_spark_context
1862+
return Column(sc._jvm.functions.array_position(_to_java_column(col), value))
1863+
1864+
18481865
@since(1.4)
18491866
def explode(col):
18501867
"""Returns a new row for each element in the given array or map.

python/pyspark/streaming/kafka.py

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -104,7 +104,8 @@ def createDirectStream(ssc, topics, kafkaParams, fromOffsets=None,
104104
:param topics: list of topic_name to consume.
105105
:param kafkaParams: Additional params for Kafka.
106106
:param fromOffsets: Per-topic/partition Kafka offsets defining the (inclusive) starting
107-
point of the stream.
107+
point of the stream (a dictionary mapping `TopicAndPartition` to
108+
integers).
108109
:param keyDecoder: A function used to decode key (default is utf8_decoder).
109110
:param valueDecoder: A function used to decode value (default is utf8_decoder).
110111
:param messageHandler: A function used to convert KafkaMessageAndMetadata. You can assess

python/pyspark/streaming/listener.py

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -23,6 +23,12 @@ class StreamingListener(object):
2323
def __init__(self):
2424
pass
2525

26+
def onStreamingStarted(self, streamingStarted):
27+
"""
28+
Called when the streaming has been started.
29+
"""
30+
pass
31+
2632
def onReceiverStarted(self, receiverStarted):
2733
"""
2834
Called when a receiver has been started

python/pyspark/streaming/tests.py

Lines changed: 7 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -507,6 +507,10 @@ def __init__(self):
507507
self.batchInfosCompleted = []
508508
self.batchInfosStarted = []
509509
self.batchInfosSubmitted = []
510+
self.streamingStartedTime = []
511+
512+
def onStreamingStarted(self, streamingStarted):
513+
self.streamingStartedTime.append(streamingStarted.time)
510514

511515
def onBatchSubmitted(self, batchSubmitted):
512516
self.batchInfosSubmitted.append(batchSubmitted.batchInfo())
@@ -530,9 +534,12 @@ def func(dstream):
530534
batchInfosSubmitted = batch_collector.batchInfosSubmitted
531535
batchInfosStarted = batch_collector.batchInfosStarted
532536
batchInfosCompleted = batch_collector.batchInfosCompleted
537+
streamingStartedTime = batch_collector.streamingStartedTime
533538

534539
self.wait_for(batchInfosCompleted, 4)
535540

541+
self.assertEqual(len(streamingStartedTime), 1)
542+
536543
self.assertGreaterEqual(len(batchInfosSubmitted), 4)
537544
for info in batchInfosSubmitted:
538545
self.assertGreaterEqual(info.batchTime().milliseconds(), 0)

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/analysis/FunctionRegistry.scala

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -402,6 +402,7 @@ object FunctionRegistry {
402402
// collection functions
403403
expression[CreateArray]("array"),
404404
expression[ArrayContains]("array_contains"),
405+
expression[ArrayPosition]("array_position"),
405406
expression[CreateMap]("map"),
406407
expression[CreateNamedStruct]("named_struct"),
407408
expression[MapKeys]("map_keys"),

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/expressions/collectionOperations.scala

Lines changed: 56 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -508,6 +508,62 @@ case class ArrayMax(child: Expression) extends UnaryExpression with ImplicitCast
508508
override def prettyName: String = "array_max"
509509
}
510510

511+
512+
/**
513+
* Returns the position of the first occurrence of element in the given array as long.
514+
* Returns 0 if the given value could not be found in the array. Returns null if either of
515+
* the arguments are null
516+
*
517+
* NOTE: that this is not zero based, but 1-based index. The first element in the array has
518+
* index 1.
519+
*/
520+
@ExpressionDescription(
521+
usage = """
522+
_FUNC_(array, element) - Returns the (1-based) index of the first element of the array as long.
523+
""",
524+
examples = """
525+
Examples:
526+
> SELECT _FUNC_(array(3, 2, 1), 1);
527+
3
528+
""",
529+
since = "2.4.0")
530+
case class ArrayPosition(left: Expression, right: Expression)
531+
extends BinaryExpression with ImplicitCastInputTypes {
532+
533+
override def dataType: DataType = LongType
534+
override def inputTypes: Seq[AbstractDataType] =
535+
Seq(ArrayType, left.dataType.asInstanceOf[ArrayType].elementType)
536+
537+
override def nullSafeEval(arr: Any, value: Any): Any = {
538+
arr.asInstanceOf[ArrayData].foreach(right.dataType, (i, v) =>
539+
if (v == value) {
540+
return (i + 1).toLong
541+
}
542+
)
543+
0L
544+
}
545+
546+
override def prettyName: String = "array_position"
547+
548+
override def doGenCode(ctx: CodegenContext, ev: ExprCode): ExprCode = {
549+
nullSafeCodeGen(ctx, ev, (arr, value) => {
550+
val pos = ctx.freshName("arrayPosition")
551+
val i = ctx.freshName("i")
552+
val getValue = CodeGenerator.getValue(arr, right.dataType, i)
553+
s"""
554+
|int $pos = 0;
555+
|for (int $i = 0; $i < $arr.numElements(); $i ++) {
556+
| if (!$arr.isNullAt($i) && ${ctx.genEqual(right.dataType, value, getValue)}) {
557+
| $pos = $i + 1;
558+
| break;
559+
| }
560+
|}
561+
|${ev.value} = (long) $pos;
562+
""".stripMargin
563+
})
564+
}
565+
}
566+
511567
/**
512568
* Transforms an array of arrays into a single array.
513569
*/

sql/catalyst/src/main/scala/org/apache/spark/sql/catalyst/util/ArrayData.scala

Lines changed: 4 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -141,28 +141,29 @@ abstract class ArrayData extends SpecializedGetters with Serializable {
141141

142142
def toArray[T: ClassTag](elementType: DataType): Array[T] = {
143143
val size = numElements()
144+
val accessor = InternalRow.getAccessor(elementType)
144145
val values = new Array[T](size)
145146
var i = 0
146147
while (i < size) {
147148
if (isNullAt(i)) {
148149
values(i) = null.asInstanceOf[T]
149150
} else {
150-
values(i) = get(i, elementType).asInstanceOf[T]
151+
values(i) = accessor(this, i).asInstanceOf[T]
151152
}
152153
i += 1
153154
}
154155
values
155156
}
156157

157-
// todo: specialize this.
158158
def foreach(elementType: DataType, f: (Int, Any) => Unit): Unit = {
159159
val size = numElements()
160+
val accessor = InternalRow.getAccessor(elementType)
160161
var i = 0
161162
while (i < size) {
162163
if (isNullAt(i)) {
163164
f(i, null)
164165
} else {
165-
f(i, get(i, elementType))
166+
f(i, accessor(this, i))
166167
}
167168
i += 1
168169
}

sql/catalyst/src/test/scala/org/apache/spark/sql/catalyst/expressions/CollectionExpressionsSuite.scala

Lines changed: 22 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -170,6 +170,28 @@ class CollectionExpressionsSuite extends SparkFunSuite with ExpressionEvalHelper
170170
checkEvaluation(Reverse(aa), Seq(Seq("e"), Seq("c", "d"), Seq("a", "b")))
171171
}
172172

173+
test("Array Position") {
174+
val a0 = Literal.create(Seq(1, null, 2, 3), ArrayType(IntegerType))
175+
val a1 = Literal.create(Seq[String](null, ""), ArrayType(StringType))
176+
val a2 = Literal.create(Seq(null), ArrayType(LongType))
177+
val a3 = Literal.create(null, ArrayType(StringType))
178+
179+
checkEvaluation(ArrayPosition(a0, Literal(3)), 4L)
180+
checkEvaluation(ArrayPosition(a0, Literal(1)), 1L)
181+
checkEvaluation(ArrayPosition(a0, Literal(0)), 0L)
182+
checkEvaluation(ArrayPosition(a0, Literal.create(null, IntegerType)), null)
183+
184+
checkEvaluation(ArrayPosition(a1, Literal("")), 2L)
185+
checkEvaluation(ArrayPosition(a1, Literal("a")), 0L)
186+
checkEvaluation(ArrayPosition(a1, Literal.create(null, StringType)), null)
187+
188+
checkEvaluation(ArrayPosition(a2, Literal(1L)), 0L)
189+
checkEvaluation(ArrayPosition(a2, Literal.create(null, LongType)), null)
190+
191+
checkEvaluation(ArrayPosition(a3, Literal("")), null)
192+
checkEvaluation(ArrayPosition(a3, Literal.create(null, StringType)), null)
193+
}
194+
173195
test("Flatten") {
174196
// Primitive-type test cases
175197
val intArrayType = ArrayType(ArrayType(IntegerType))

sql/core/src/main/scala/org/apache/spark/sql/functions.scala

Lines changed: 14 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -3038,6 +3038,20 @@ object functions {
30383038
ArrayContains(column.expr, Literal(value))
30393039
}
30403040

3041+
/**
3042+
* Locates the position of the first occurrence of the value in the given array as long.
3043+
* Returns null if either of the arguments are null.
3044+
*
3045+
* @note The position is not zero based, but 1 based index. Returns 0 if value
3046+
* could not be found in array.
3047+
*
3048+
* @group collection_funcs
3049+
* @since 2.4.0
3050+
*/
3051+
def array_position(column: Column, value: Any): Column = withExpr {
3052+
ArrayPosition(column.expr, Literal(value))
3053+
}
3054+
30413055
/**
30423056
* Creates a new row for each element in the given array or map column.
30433057
*

sql/core/src/test/scala/org/apache/spark/sql/DataFrameFunctionsSuite.scala

Lines changed: 34 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -535,6 +535,40 @@ class DataFrameFunctionsSuite extends QueryTest with SharedSQLContext {
535535
}
536536
}
537537

538+
test("array position function") {
539+
val df = Seq(
540+
(Seq[Int](1, 2), "x"),
541+
(Seq[Int](), "x")
542+
).toDF("a", "b")
543+
544+
checkAnswer(
545+
df.select(array_position(df("a"), 1)),
546+
Seq(Row(1L), Row(0L))
547+
)
548+
checkAnswer(
549+
df.selectExpr("array_position(a, 1)"),
550+
Seq(Row(1L), Row(0L))
551+
)
552+
553+
checkAnswer(
554+
df.select(array_position(df("a"), null)),
555+
Seq(Row(null), Row(null))
556+
)
557+
checkAnswer(
558+
df.selectExpr("array_position(a, null)"),
559+
Seq(Row(null), Row(null))
560+
)
561+
562+
checkAnswer(
563+
df.selectExpr("array_position(array(array(1), null)[0], 1)"),
564+
Seq(Row(1L), Row(1L))
565+
)
566+
checkAnswer(
567+
df.selectExpr("array_position(array(1, null), array(1, null)[0])"),
568+
Seq(Row(1L), Row(1L))
569+
)
570+
}
571+
538572
test("flatten function") {
539573
val dummyFilter = (c: Column) => c.isNull || c.isNotNull // to switch codeGen on
540574
val oneRowDF = Seq((1, "a", Seq(1, 2, 3))).toDF("i", "s", "arr")

0 commit comments

Comments
 (0)