1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
use std::any::Any;
use std::pin::Pin;
use std::sync::Arc;
use std::task::{Context, Poll};
use crate::error::{DataFusionError, Result};
use crate::physical_plan::{ExecutionPlan, Partitioning, PhysicalExpr};
use arrow::datatypes::{Field, Schema, SchemaRef};
use arrow::error::Result as ArrowResult;
use arrow::record_batch::RecordBatch;
use super::{RecordBatchStream, SendableRecordBatchStream};
use async_trait::async_trait;
use futures::stream::Stream;
use futures::stream::StreamExt;
#[derive(Debug)]
pub struct ProjectionExec {
expr: Vec<(Arc<dyn PhysicalExpr>, String)>,
schema: SchemaRef,
input: Arc<dyn ExecutionPlan>,
}
impl ProjectionExec {
pub fn try_new(
expr: Vec<(Arc<dyn PhysicalExpr>, String)>,
input: Arc<dyn ExecutionPlan>,
) -> Result<Self> {
let input_schema = input.schema();
let fields: Result<Vec<_>> = expr
.iter()
.map(|(e, name)| {
Ok(Field::new(
name,
e.data_type(&input_schema)?,
e.nullable(&input_schema)?,
))
})
.collect();
let schema = Arc::new(Schema::new(fields?));
Ok(Self {
expr,
schema,
input: input.clone(),
})
}
pub fn expr(&self) -> &[(Arc<dyn PhysicalExpr>, String)] {
&self.expr
}
pub fn input(&self) -> &Arc<dyn ExecutionPlan> {
&self.input
}
}
#[async_trait]
impl ExecutionPlan for ProjectionExec {
fn as_any(&self) -> &dyn Any {
self
}
fn schema(&self) -> SchemaRef {
self.schema.clone()
}
fn children(&self) -> Vec<Arc<dyn ExecutionPlan>> {
vec![self.input.clone()]
}
fn output_partitioning(&self) -> Partitioning {
self.input.output_partitioning()
}
fn with_new_children(
&self,
children: Vec<Arc<dyn ExecutionPlan>>,
) -> Result<Arc<dyn ExecutionPlan>> {
match children.len() {
1 => Ok(Arc::new(ProjectionExec::try_new(
self.expr.clone(),
children[0].clone(),
)?)),
_ => Err(DataFusionError::Internal(
"ProjectionExec wrong number of children".to_string(),
)),
}
}
async fn execute(&self, partition: usize) -> Result<SendableRecordBatchStream> {
Ok(Box::pin(ProjectionStream {
schema: self.schema.clone(),
expr: self.expr.iter().map(|x| x.0.clone()).collect(),
input: self.input.execute(partition).await?,
}))
}
}
fn batch_project(
batch: &RecordBatch,
expressions: &[Arc<dyn PhysicalExpr>],
schema: &SchemaRef,
) -> ArrowResult<RecordBatch> {
expressions
.iter()
.map(|expr| expr.evaluate(&batch))
.map(|r| r.map(|v| v.into_array(batch.num_rows())))
.collect::<Result<Vec<_>>>()
.map_or_else(
|e| Err(DataFusionError::into_arrow_external_error(e)),
|arrays| RecordBatch::try_new(schema.clone(), arrays),
)
}
struct ProjectionStream {
schema: SchemaRef,
expr: Vec<Arc<dyn PhysicalExpr>>,
input: SendableRecordBatchStream,
}
impl Stream for ProjectionStream {
type Item = ArrowResult<RecordBatch>;
fn poll_next(
mut self: Pin<&mut Self>,
cx: &mut Context<'_>,
) -> Poll<Option<Self::Item>> {
self.input.poll_next_unpin(cx).map(|x| match x {
Some(Ok(batch)) => Some(batch_project(&batch, &self.expr, &self.schema)),
other => other,
})
}
fn size_hint(&self) -> (usize, Option<usize>) {
self.input.size_hint()
}
}
impl RecordBatchStream for ProjectionStream {
fn schema(&self) -> SchemaRef {
self.schema.clone()
}
}
#[cfg(test)]
mod tests {
use super::*;
use crate::physical_plan::csv::{CsvExec, CsvReadOptions};
use crate::physical_plan::expressions::col;
use crate::test;
use futures::future;
#[tokio::test]
async fn project_first_column() -> Result<()> {
let schema = test::aggr_test_schema();
let partitions = 4;
let path = test::create_partitioned_csv("aggregate_test_100.csv", partitions)?;
let csv = CsvExec::try_new(
&path,
CsvReadOptions::new().schema(&schema),
None,
1024,
None,
)?;
let projection =
ProjectionExec::try_new(vec![(col("c1"), "c1".to_string())], Arc::new(csv))?;
let mut partition_count = 0;
let mut row_count = 0;
for partition in 0..projection.output_partitioning().partition_count() {
partition_count += 1;
let stream = projection.execute(partition).await?;
row_count += stream
.map(|batch| {
let batch = batch.unwrap();
assert_eq!(1, batch.num_columns());
batch.num_rows()
})
.fold(0, |acc, x| future::ready(acc + x))
.await;
}
assert_eq!(partitions, partition_count);
assert_eq!(100, row_count);
Ok(())
}
}