Skip to content

Commit 52da49a

Browse files
committed
fetch results for Trino more efficiently
1 parent 6979c65 commit 52da49a

File tree

2 files changed

+35
-8
lines changed

2 files changed

+35
-8
lines changed

connectorx/src/sources/trino/mod.rs

Lines changed: 33 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -129,7 +129,6 @@ where
129129
fn fetch_metadata(&mut self) {
130130
assert!(!self.queries.is_empty());
131131

132-
// TODO: prevent from running the same query multiple times (limit1 + no limit)
133132
let first_query = &self.queries[0];
134133
let cxq = limit1_query(first_query, &GenericDialect {})?;
135134

@@ -238,6 +237,9 @@ impl SourcePartition for TrinoSourcePartition {
238237
}
239238

240239
pub struct TrinoSourcePartitionParser<'a> {
240+
rt: Arc<Runtime>,
241+
client: Arc<Client>,
242+
next_uri: Option<String>,
241243
rows: Vec<Row>,
242244
ncols: usize,
243245
current_col: usize,
@@ -253,11 +255,19 @@ impl<'a> TrinoSourcePartitionParser<'a> {
253255
query: CXQuery,
254256
schema: &[TrinoTypeSystem],
255257
) -> Self {
256-
let rows = client.get_all::<Row>(query.to_string());
257-
let data = rt.block_on(rows).map_err(TrinoSourceError::PrustoError)?;
258-
let rows = data.clone().into_vec();
258+
let results = rt
259+
.block_on(client.get::<Row>(query.to_string()))
260+
.map_err(TrinoSourceError::PrustoError)?;
261+
262+
let rows = match results.data_set {
263+
Some(x) => x.into_vec(),
264+
_ => vec![],
265+
};
259266

260267
Self {
268+
rt,
269+
client,
270+
next_uri: results.next_uri,
261271
rows,
262272
ncols: schema.len(),
263273
current_row: 0,
@@ -283,8 +293,25 @@ impl<'a> PartitionParser<'a> for TrinoSourcePartitionParser<'a> {
283293
fn fetch_next(&mut self) -> (usize, bool) {
284294
assert!(self.current_col == 0);
285295

286-
// results are always fetched in a single batch for Prusto
287-
(self.rows.len(), true)
296+
match self.next_uri.clone() {
297+
Some(uri) => {
298+
let results = self
299+
.rt
300+
.block_on(self.client.get_next::<Row>(&uri))
301+
.map_err(TrinoSourceError::PrustoError)?;
302+
303+
self.rows = match results.data_set {
304+
Some(x) => x.into_vec(),
305+
_ => vec![],
306+
};
307+
308+
self.current_row = 0;
309+
self.next_uri = results.next_uri;
310+
311+
(self.rows.len(), false)
312+
}
313+
None => return (self.rows.len(), true),
314+
}
288315
}
289316
}
290317

connectorx/src/sources/trino/typesystem.rs

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -1,10 +1,10 @@
11
use super::errors::TrinoSourceError;
22
use chrono::{NaiveDate, NaiveDateTime, NaiveTime};
33
use fehler::{throw, throws};
4-
use prusto::{Presto, PrestoFloat, PrestoInt, PrestoTy};
4+
use prusto::{PrestoFloat, PrestoInt, PrestoTy};
55
use std::convert::TryFrom;
66

7-
// TODO: implement Tuple, Row, Array and Map as well as UUID
7+
// TODO: implement Tuple, Row, Array and Map
88
#[derive(Copy, Clone, Debug, PartialEq)]
99
pub enum TrinoTypeSystem {
1010
Date(bool),

0 commit comments

Comments
 (0)