arrow_integration_testing/flight_server_scenarios/
integration_test.rs

1// Licensed to the Apache Software Foundation (ASF) under one
2// or more contributor license agreements.  See the NOTICE file
3// distributed with this work for additional information
4// regarding copyright ownership.  The ASF licenses this file
5// to you under the Apache License, Version 2.0 (the
6// "License"); you may not use this file except in compliance
7// with the License.  You may obtain a copy of the License at
8//
9//   http://www.apache.org/licenses/LICENSE-2.0
10//
11// Unless required by applicable law or agreed to in writing,
12// software distributed under the License is distributed on an
13// "AS IS" BASIS, WITHOUT WARRANTIES OR CONDITIONS OF ANY
14// KIND, either express or implied.  See the License for the
15// specific language governing permissions and limitations
16// under the License.
17
18//! Integration tests for the Flight server.
19
20use core::str;
21use std::collections::HashMap;
22use std::pin::Pin;
23use std::sync::Arc;
24
25use arrow::{
26    array::ArrayRef,
27    buffer::Buffer,
28    datatypes::Schema,
29    datatypes::SchemaRef,
30    ipc::{self, reader, writer},
31    record_batch::RecordBatch,
32};
33use arrow_flight::{
34    flight_descriptor::DescriptorType, flight_service_server::FlightService,
35    flight_service_server::FlightServiceServer, Action, ActionType, Criteria, Empty, FlightData,
36    FlightDescriptor, FlightEndpoint, FlightInfo, HandshakeRequest, HandshakeResponse, IpcMessage,
37    PollInfo, PutResult, SchemaAsIpc, SchemaResult, Ticket,
38};
39use futures::{channel::mpsc, sink::SinkExt, Stream, StreamExt};
40use tokio::sync::Mutex;
41use tonic::{transport::Server, Request, Response, Status, Streaming};
42
43type TonicStream<T> = Pin<Box<dyn Stream<Item = T> + Send + Sync + 'static>>;
44
45type Error = Box<dyn std::error::Error + Send + Sync + 'static>;
46type Result<T = (), E = Error> = std::result::Result<T, E>;
47
48/// Run a scenario that tests integration testing.
49pub async fn scenario_setup(port: u16) -> Result {
50    let addr = super::listen_on(port).await?;
51    let resolved_port = addr.port();
52
53    let service = FlightServiceImpl {
54        // See https://github.com/apache/arrow-rs/issues/6577
55        // C# had trouble resolving addressed like 0.0.0.0:port
56        // server_location: format!("grpc+tcp://{addr}"),
57        server_location: format!("grpc+tcp://localhost:{resolved_port}"),
58        ..Default::default()
59    };
60    let svc = FlightServiceServer::new(service);
61
62    let server = Server::builder().add_service(svc).serve(addr);
63
64    // NOTE: Log output used in tests to signal server is ready
65    println!("Server listening on localhost:{}", addr.port());
66    server.await?;
67    Ok(())
68}
69
70#[derive(Debug, Clone)]
71struct IntegrationDataset {
72    schema: Schema,
73    chunks: Vec<RecordBatch>,
74}
75
76/// Flight service implementation for integration testing
77#[derive(Clone, Default)]
78pub struct FlightServiceImpl {
79    server_location: String,
80    uploaded_chunks: Arc<Mutex<HashMap<String, IntegrationDataset>>>,
81}
82
83impl FlightServiceImpl {
84    fn endpoint_from_path(&self, path: &str) -> FlightEndpoint {
85        super::endpoint(path, &self.server_location)
86    }
87}
88
89#[tonic::async_trait]
90impl FlightService for FlightServiceImpl {
91    type HandshakeStream = TonicStream<Result<HandshakeResponse, Status>>;
92    type ListFlightsStream = TonicStream<Result<FlightInfo, Status>>;
93    type DoGetStream = TonicStream<Result<FlightData, Status>>;
94    type DoPutStream = TonicStream<Result<PutResult, Status>>;
95    type DoActionStream = TonicStream<Result<arrow_flight::Result, Status>>;
96    type ListActionsStream = TonicStream<Result<ActionType, Status>>;
97    type DoExchangeStream = TonicStream<Result<FlightData, Status>>;
98
99    async fn get_schema(
100        &self,
101        _request: Request<FlightDescriptor>,
102    ) -> Result<Response<SchemaResult>, Status> {
103        Err(Status::unimplemented("Not yet implemented"))
104    }
105
106    async fn do_get(
107        &self,
108        request: Request<Ticket>,
109    ) -> Result<Response<Self::DoGetStream>, Status> {
110        let ticket = request.into_inner();
111
112        let key = str::from_utf8(&ticket.ticket)
113            .map_err(|e| Status::invalid_argument(format!("Invalid ticket: {e:?}")))?;
114
115        let uploaded_chunks = self.uploaded_chunks.lock().await;
116
117        let flight = uploaded_chunks
118            .get(key)
119            .ok_or_else(|| Status::not_found(format!("Could not find flight. {key}")))?;
120
121        let options = arrow::ipc::writer::IpcWriteOptions::default();
122        let mut dictionary_tracker = writer::DictionaryTracker::new(false);
123        let data_gen = writer::IpcDataGenerator::default();
124        let data = IpcMessage(
125            data_gen
126                .schema_to_bytes_with_dictionary_tracker(
127                    &flight.schema,
128                    &mut dictionary_tracker,
129                    &options,
130                )
131                .ipc_message
132                .into(),
133        );
134        let schema_flight_data = FlightData {
135            data_header: data.0,
136            ..Default::default()
137        };
138
139        let schema = std::iter::once(Ok(schema_flight_data));
140
141        let batches = flight
142            .chunks
143            .iter()
144            .enumerate()
145            .flat_map(|(counter, batch)| {
146                let (encoded_dictionaries, encoded_batch) = data_gen
147                    .encoded_batch(batch, &mut dictionary_tracker, &options)
148                    .expect("DictionaryTracker configured above to not error on replacement");
149
150                let dictionary_flight_data = encoded_dictionaries.into_iter().map(Into::into);
151                let mut batch_flight_data: FlightData = encoded_batch.into();
152
153                // Only the record batch's FlightData gets app_metadata
154                let metadata = counter.to_string().into();
155                batch_flight_data.app_metadata = metadata;
156
157                dictionary_flight_data
158                    .chain(std::iter::once(batch_flight_data))
159                    .map(Ok)
160            });
161
162        let output = futures::stream::iter(schema.chain(batches).collect::<Vec<_>>());
163
164        Ok(Response::new(Box::pin(output) as Self::DoGetStream))
165    }
166
167    async fn handshake(
168        &self,
169        _request: Request<Streaming<HandshakeRequest>>,
170    ) -> Result<Response<Self::HandshakeStream>, Status> {
171        Err(Status::unimplemented("Not yet implemented"))
172    }
173
174    async fn list_flights(
175        &self,
176        _request: Request<Criteria>,
177    ) -> Result<Response<Self::ListFlightsStream>, Status> {
178        Err(Status::unimplemented("Not yet implemented"))
179    }
180
181    async fn get_flight_info(
182        &self,
183        request: Request<FlightDescriptor>,
184    ) -> Result<Response<FlightInfo>, Status> {
185        let descriptor = request.into_inner();
186
187        match descriptor.r#type {
188            t if t == DescriptorType::Path as i32 => {
189                let path = &descriptor.path;
190                if path.is_empty() {
191                    return Err(Status::invalid_argument("Invalid path"));
192                }
193
194                let uploaded_chunks = self.uploaded_chunks.lock().await;
195                let flight = uploaded_chunks.get(&path[0]).ok_or_else(|| {
196                    Status::not_found(format!("Could not find flight. {}", path[0]))
197                })?;
198
199                let endpoint = self.endpoint_from_path(&path[0]);
200
201                let total_records: usize = flight.chunks.iter().map(|chunk| chunk.num_rows()).sum();
202
203                let options = arrow::ipc::writer::IpcWriteOptions::default();
204                let message = SchemaAsIpc::new(&flight.schema, &options)
205                    .try_into()
206                    .expect(
207                        "Could not generate schema bytes from schema stored by a DoPut; \
208                         this should be impossible",
209                    );
210                let IpcMessage(schema) = message;
211
212                let info = FlightInfo {
213                    schema,
214                    flight_descriptor: Some(descriptor.clone()),
215                    endpoint: vec![endpoint],
216                    total_records: total_records as i64,
217                    total_bytes: -1,
218                    ordered: false,
219                    app_metadata: vec![].into(),
220                };
221
222                Ok(Response::new(info))
223            }
224            other => Err(Status::unimplemented(format!("Request type: {other}"))),
225        }
226    }
227
228    async fn poll_flight_info(
229        &self,
230        _request: Request<FlightDescriptor>,
231    ) -> Result<Response<PollInfo>, Status> {
232        Err(Status::unimplemented("Not yet implemented"))
233    }
234
235    async fn do_put(
236        &self,
237        request: Request<Streaming<FlightData>>,
238    ) -> Result<Response<Self::DoPutStream>, Status> {
239        let mut input_stream = request.into_inner();
240        let flight_data = input_stream
241            .message()
242            .await?
243            .ok_or_else(|| Status::invalid_argument("Must send some FlightData"))?;
244
245        let descriptor = flight_data
246            .flight_descriptor
247            .clone()
248            .ok_or_else(|| Status::invalid_argument("Must have a descriptor"))?;
249
250        if descriptor.r#type != DescriptorType::Path as i32 || descriptor.path.is_empty() {
251            return Err(Status::invalid_argument("Must specify a path"));
252        }
253
254        let key = descriptor.path[0].clone();
255
256        let schema = Schema::try_from(&flight_data)
257            .map_err(|e| Status::invalid_argument(format!("Invalid schema: {e:?}")))?;
258        let schema_ref = Arc::new(schema.clone());
259
260        let (response_tx, response_rx) = mpsc::channel(10);
261
262        let uploaded_chunks = self.uploaded_chunks.clone();
263
264        tokio::spawn(async {
265            let mut error_tx = response_tx.clone();
266            if let Err(e) = save_uploaded_chunks(
267                uploaded_chunks,
268                schema_ref,
269                input_stream,
270                response_tx,
271                schema,
272                key,
273            )
274            .await
275            {
276                error_tx.send(Err(e)).await.expect("Error sending error")
277            }
278        });
279
280        Ok(Response::new(Box::pin(response_rx) as Self::DoPutStream))
281    }
282
283    async fn do_action(
284        &self,
285        _request: Request<Action>,
286    ) -> Result<Response<Self::DoActionStream>, Status> {
287        Err(Status::unimplemented("Not yet implemented"))
288    }
289
290    async fn list_actions(
291        &self,
292        _request: Request<Empty>,
293    ) -> Result<Response<Self::ListActionsStream>, Status> {
294        Err(Status::unimplemented("Not yet implemented"))
295    }
296
297    async fn do_exchange(
298        &self,
299        _request: Request<Streaming<FlightData>>,
300    ) -> Result<Response<Self::DoExchangeStream>, Status> {
301        Err(Status::unimplemented("Not yet implemented"))
302    }
303}
304
305async fn send_app_metadata(
306    tx: &mut mpsc::Sender<Result<PutResult, Status>>,
307    app_metadata: &[u8],
308) -> Result<(), Status> {
309    tx.send(Ok(PutResult {
310        app_metadata: app_metadata.to_vec().into(),
311    }))
312    .await
313    .map_err(|e| Status::internal(format!("Could not send PutResult: {e:?}")))
314}
315
316async fn record_batch_from_message(
317    message: ipc::Message<'_>,
318    data_body: &Buffer,
319    schema_ref: SchemaRef,
320    dictionaries_by_id: &HashMap<i64, ArrayRef>,
321) -> Result<RecordBatch, Status> {
322    let ipc_batch = message
323        .header_as_record_batch()
324        .ok_or_else(|| Status::internal("Could not parse message header as record batch"))?;
325
326    let arrow_batch_result = reader::read_record_batch(
327        data_body,
328        ipc_batch,
329        schema_ref,
330        dictionaries_by_id,
331        None,
332        &message.version(),
333    );
334
335    arrow_batch_result
336        .map_err(|e| Status::internal(format!("Could not convert to RecordBatch: {e:?}")))
337}
338
339async fn dictionary_from_message(
340    message: ipc::Message<'_>,
341    data_body: &Buffer,
342    schema_ref: SchemaRef,
343    dictionaries_by_id: &mut HashMap<i64, ArrayRef>,
344) -> Result<(), Status> {
345    let ipc_batch = message
346        .header_as_dictionary_batch()
347        .ok_or_else(|| Status::internal("Could not parse message header as dictionary batch"))?;
348
349    let dictionary_batch_result = reader::read_dictionary(
350        data_body,
351        ipc_batch,
352        &schema_ref,
353        dictionaries_by_id,
354        &message.version(),
355    );
356    dictionary_batch_result
357        .map_err(|e| Status::internal(format!("Could not convert to Dictionary: {e:?}")))
358}
359
360async fn save_uploaded_chunks(
361    uploaded_chunks: Arc<Mutex<HashMap<String, IntegrationDataset>>>,
362    schema_ref: Arc<Schema>,
363    mut input_stream: Streaming<FlightData>,
364    mut response_tx: mpsc::Sender<Result<PutResult, Status>>,
365    schema: Schema,
366    key: String,
367) -> Result<(), Status> {
368    let mut chunks = vec![];
369    let mut uploaded_chunks = uploaded_chunks.lock().await;
370
371    let mut dictionaries_by_id = HashMap::new();
372
373    while let Some(Ok(data)) = input_stream.next().await {
374        let message = arrow::ipc::root_as_message(&data.data_header[..])
375            .map_err(|e| Status::internal(format!("Could not parse message: {e:?}")))?;
376
377        match message.header_type() {
378            ipc::MessageHeader::Schema => {
379                return Err(Status::internal(
380                    "Not expecting a schema when messages are read",
381                ))
382            }
383            ipc::MessageHeader::RecordBatch => {
384                send_app_metadata(&mut response_tx, &data.app_metadata).await?;
385
386                let batch = record_batch_from_message(
387                    message,
388                    &Buffer::from(data.data_body.as_ref()),
389                    schema_ref.clone(),
390                    &dictionaries_by_id,
391                )
392                .await?;
393
394                chunks.push(batch);
395            }
396            ipc::MessageHeader::DictionaryBatch => {
397                dictionary_from_message(
398                    message,
399                    &Buffer::from(data.data_body.as_ref()),
400                    schema_ref.clone(),
401                    &mut dictionaries_by_id,
402                )
403                .await?;
404            }
405            t => {
406                return Err(Status::internal(format!(
407                    "Reading types other than record batches not yet supported, \
408                                              unable to read {t:?}"
409                )));
410            }
411        }
412    }
413
414    let dataset = IntegrationDataset { schema, chunks };
415    uploaded_chunks.insert(key, dataset);
416
417    Ok(())
418}