@@ -30,6 +30,7 @@ use common_types::{
30
30
column_block:: { ColumnBlock , ColumnBlockBuilder } ,
31
31
column_schema:: ColumnId ,
32
32
datum:: Datum ,
33
+ record_batch:: RecordBatch as CommonRecordBatch ,
33
34
row:: { Row , RowBuilder , RowGroup } ,
34
35
schema:: Schema ,
35
36
} ;
@@ -54,12 +55,15 @@ use query_frontend::{
54
55
} ;
55
56
use runtime:: Priority ;
56
57
use snafu:: { ensure, OptionExt , ResultExt , Snafu } ;
57
- use table_engine:: table:: { TableRef , WriteRequest } ;
58
+ use table_engine:: {
59
+ stream:: SendableRecordBatchStream ,
60
+ table:: { TableRef , WriteRequest } ,
61
+ } ;
62
+ use tokio:: sync:: mpsc;
58
63
59
64
use crate :: {
60
65
context:: Context ,
61
66
interpreter:: { Insert , Interpreter , InterpreterPtr , Output , Result as InterpreterResult } ,
62
- RecordBatchVec ,
63
67
} ;
64
68
65
69
#[ derive( Debug , Snafu ) ]
@@ -115,10 +119,23 @@ pub enum Error {
115
119
116
120
#[ snafu( display( "Record columns not enough, len:{}, index:{}" , len, index) ) ]
117
121
RecordColumnsNotEnough { len : usize , index : usize } ,
122
+
123
+ #[ snafu( display( "Failed to do select, err:{}" , source) ) ]
124
+ Select { source : table_engine:: stream:: Error } ,
125
+
126
+ #[ snafu( display( "Failed to send msg in channel, err:{}" , msg) ) ]
127
+ MsgChannel { msg : String } ,
128
+
129
+ #[ snafu( display( "Failed to join async task, err:{}" , msg) ) ]
130
+ AsyncTask { msg : String } ,
118
131
}
119
132
120
133
define_result ! ( Error ) ;
121
134
135
+ // TODO: make those configurable
136
+ const INSERT_SELECT_ROW_BATCH_NUM : usize = 1000 ;
137
+ const INSERT_SELECT_PENDING_BATCH_NUM : usize = 3 ;
138
+
122
139
pub struct InsertInterpreter {
123
140
ctx : Context ,
124
141
plan : InsertPlan ,
@@ -152,14 +169,18 @@ impl Interpreter for InsertInterpreter {
152
169
default_value_map,
153
170
} = self . plan ;
154
171
155
- let mut rows = match source {
156
- InsertSource :: Values { row_group } => row_group,
172
+ match source {
173
+ InsertSource :: Values { row_group : rows } => {
174
+ let num_rows =
175
+ prepare_and_write_table ( table. clone ( ) , rows, & default_value_map) . await ?;
176
+
177
+ Ok ( Output :: AffectedRows ( num_rows) )
178
+ }
157
179
InsertSource :: Select {
158
180
query : query_plan,
159
181
column_index_in_insert,
160
182
} => {
161
- // TODO: support streaming insert
162
- let record_batches = exec_select_logical_plan (
183
+ let mut record_batches_stream = exec_select_logical_plan (
163
184
self . ctx ,
164
185
query_plan,
165
186
self . executor ,
@@ -168,38 +189,120 @@ impl Interpreter for InsertInterpreter {
168
189
. await
169
190
. context ( Insert ) ?;
170
191
171
- if record_batches. is_empty ( ) {
172
- return Ok ( Output :: AffectedRows ( 0 ) ) ;
173
- }
192
+ let ( tx, rx) = mpsc:: channel ( INSERT_SELECT_PENDING_BATCH_NUM ) ;
193
+ let producer = tokio:: spawn ( async move {
194
+ while let Some ( record_batch) = record_batches_stream
195
+ . try_next ( )
196
+ . await
197
+ . context ( Select )
198
+ . context ( Insert ) ?
199
+ {
200
+ if record_batch. is_empty ( ) {
201
+ continue ;
202
+ }
203
+ if let Err ( e) = tx. send ( record_batch) . await {
204
+ return Err ( Error :: MsgChannel {
205
+ msg : format ! ( "{}" , e) ,
206
+ } )
207
+ . context ( Insert ) ?;
208
+ }
209
+ }
210
+ Ok ( ( ) )
211
+ } ) ;
212
+
213
+ let consumer = tokio:: spawn ( async move {
214
+ let mut rx = rx;
215
+ let mut result_rows = 0 ;
216
+ let mut pending_rows = 0 ;
217
+ let mut record_batches = Vec :: new ( ) ;
218
+ while let Some ( record_batch) = rx. recv ( ) . await {
219
+ pending_rows += record_batch. num_rows ( ) ;
220
+ record_batches. push ( record_batch) ;
221
+ if pending_rows >= INSERT_SELECT_ROW_BATCH_NUM {
222
+ pending_rows = 0 ;
223
+ let num_rows = write_record_batches (
224
+ & mut record_batches,
225
+ column_index_in_insert. as_slice ( ) ,
226
+ table. clone ( ) ,
227
+ & default_value_map,
228
+ )
229
+ . await ?;
230
+ result_rows += num_rows;
231
+ }
232
+ }
174
233
175
- convert_records_to_row_group ( record_batches, column_index_in_insert, table. schema ( ) )
176
- . context ( Insert ) ?
234
+ if !record_batches. is_empty ( ) {
235
+ let num_rows = write_record_batches (
236
+ & mut record_batches,
237
+ column_index_in_insert. as_slice ( ) ,
238
+ table,
239
+ & default_value_map,
240
+ )
241
+ . await ?;
242
+ result_rows += num_rows;
243
+ }
244
+ Ok ( result_rows)
245
+ } ) ;
246
+
247
+ match tokio:: try_join!( producer, consumer) {
248
+ Ok ( ( select_res, write_rows) ) => {
249
+ select_res?;
250
+ Ok ( Output :: AffectedRows ( write_rows?) )
251
+ }
252
+ Err ( e) => Err ( Error :: AsyncTask {
253
+ msg : format ! ( "{}" , e) ,
254
+ } )
255
+ . context ( Insert ) ?,
256
+ }
177
257
}
178
- } ;
258
+ }
259
+ }
260
+ }
179
261
180
- maybe_generate_tsid ( & mut rows) . context ( Insert ) ?;
262
+ async fn write_record_batches (
263
+ record_batches : & mut Vec < CommonRecordBatch > ,
264
+ column_index_in_insert : & [ InsertMode ] ,
265
+ table : TableRef ,
266
+ default_value_map : & BTreeMap < usize , DfLogicalExpr > ,
267
+ ) -> InterpreterResult < usize > {
268
+ let row_group = convert_records_to_row_group (
269
+ record_batches. as_slice ( ) ,
270
+ column_index_in_insert,
271
+ table. schema ( ) ,
272
+ )
273
+ . context ( Insert ) ?;
274
+ record_batches. clear ( ) ;
275
+
276
+ prepare_and_write_table ( table, row_group, default_value_map) . await
277
+ }
181
278
182
- // Fill default values
183
- fill_default_values ( table. clone ( ) , & mut rows, & default_value_map) . context ( Insert ) ?;
279
+ async fn prepare_and_write_table (
280
+ table : TableRef ,
281
+ mut row_group : RowGroup ,
282
+ default_value_map : & BTreeMap < usize , DfLogicalExpr > ,
283
+ ) -> InterpreterResult < usize > {
284
+ maybe_generate_tsid ( & mut row_group) . context ( Insert ) ?;
184
285
185
- let request = WriteRequest { row_group : rows } ;
286
+ // Fill default values
287
+ fill_default_values ( table. clone ( ) , & mut row_group, default_value_map) . context ( Insert ) ?;
186
288
187
- let num_rows = table
188
- . write ( request)
189
- . await
190
- . context ( WriteTable )
191
- . context ( Insert ) ?;
289
+ let request = WriteRequest { row_group } ;
192
290
193
- Ok ( Output :: AffectedRows ( num_rows) )
194
- }
291
+ let num_rows = table
292
+ . write ( request)
293
+ . await
294
+ . context ( WriteTable )
295
+ . context ( Insert ) ?;
296
+
297
+ Ok ( num_rows)
195
298
}
196
299
197
300
async fn exec_select_logical_plan (
198
301
ctx : Context ,
199
302
query_plan : QueryPlan ,
200
303
executor : ExecutorRef ,
201
304
physical_planner : PhysicalPlannerRef ,
202
- ) -> Result < RecordBatchVec > {
305
+ ) -> Result < SendableRecordBatchStream > {
203
306
let priority = Priority :: High ;
204
307
205
308
let query_ctx = ctx
@@ -216,34 +319,25 @@ async fn exec_select_logical_plan(
216
319
} ) ?;
217
320
218
321
// Execute select physical plan.
219
- let record_batch_stream = executor
322
+ let record_batch_stream: SendableRecordBatchStream = executor
220
323
. execute ( & query_ctx, physical_plan)
221
324
. await
222
325
. box_err ( )
223
326
. context ( ExecuteSelectPlan {
224
327
msg : "failed to execute select physical plan" ,
225
328
} ) ?;
226
329
227
- let record_batches =
228
- record_batch_stream
229
- . try_collect ( )
230
- . await
231
- . box_err ( )
232
- . context ( ExecuteSelectPlan {
233
- msg : "failed to collect select execution results" ,
234
- } ) ?;
235
-
236
- Ok ( record_batches)
330
+ Ok ( record_batch_stream)
237
331
}
238
332
239
333
fn convert_records_to_row_group (
240
- record_batches : RecordBatchVec ,
241
- column_index_in_insert : Vec < InsertMode > ,
334
+ record_batches : & [ CommonRecordBatch ] ,
335
+ column_index_in_insert : & [ InsertMode ] ,
242
336
schema : Schema ,
243
337
) -> Result < RowGroup > {
244
338
let mut data_rows: Vec < Row > = Vec :: new ( ) ;
245
339
246
- for record in & record_batches {
340
+ for record in record_batches {
247
341
let num_cols = record. num_columns ( ) ;
248
342
let num_rows = record. num_rows ( ) ;
249
343
for row_idx in 0 ..num_rows {
0 commit comments