17
17
18
18
use anyhow:: Context ;
19
19
use ballista_core:: config:: TaskSchedulingPolicy ;
20
- use ballista_core:: error:: BallistaError ;
21
20
use ballista_core:: execution_plans:: ShuffleWriterExec ;
22
21
use ballista_core:: serde:: protobuf:: execute_query_params:: Query ;
23
22
use ballista_core:: serde:: protobuf:: executor_grpc_client:: ExecutorGrpcClient ;
@@ -26,30 +25,27 @@ use ballista_core::serde::protobuf::scheduler_grpc_server::SchedulerGrpc;
26
25
use ballista_core:: serde:: protobuf:: {
27
26
job_status, ExecuteQueryParams , ExecuteQueryResult , ExecutorHeartbeat , FailedJob ,
28
27
FileType , GetFileMetadataParams , GetFileMetadataResult , GetJobStatusParams ,
29
- GetJobStatusResult , HeartBeatParams , HeartBeatResult , JobStatus , PartitionId ,
30
- PollWorkParams , PollWorkResult , QueuedJob , RegisterExecutorParams ,
31
- RegisterExecutorResult , RunningJob , TaskDefinition , TaskStatus ,
32
- UpdateTaskStatusParams , UpdateTaskStatusResult ,
28
+ GetJobStatusResult , HeartBeatParams , HeartBeatResult , JobStatus , PollWorkParams ,
29
+ PollWorkResult , QueuedJob , RegisterExecutorParams , RegisterExecutorResult ,
30
+ TaskDefinition , UpdateTaskStatusParams , UpdateTaskStatusResult ,
33
31
} ;
34
32
use ballista_core:: serde:: scheduler:: to_proto:: hash_partitioning_to_proto;
35
33
use ballista_core:: serde:: scheduler:: { ExecutorData , ExecutorMetadata } ;
36
34
use ballista_core:: serde:: { AsExecutionPlan , AsLogicalPlan } ;
37
35
use datafusion:: datasource:: file_format:: parquet:: ParquetFormat ;
38
36
use datafusion:: datasource:: file_format:: FileFormat ;
39
37
use datafusion:: datasource:: object_store:: { local:: LocalFileSystem , ObjectStore } ;
40
- use datafusion:: logical_plan:: LogicalPlan ;
41
- use datafusion:: physical_plan:: ExecutionPlan ;
42
38
use futures:: StreamExt ;
43
39
use log:: { debug, error, info, trace, warn} ;
44
40
use rand:: { distributions:: Alphanumeric , thread_rng, Rng } ;
45
41
use std:: collections:: HashSet ;
46
42
use std:: convert:: TryInto ;
47
43
use std:: sync:: Arc ;
48
- use std:: time:: { Instant , SystemTime , UNIX_EPOCH } ;
44
+ use std:: time:: { SystemTime , UNIX_EPOCH } ;
49
45
use tonic:: { Request , Response , Status } ;
50
46
51
- use crate :: planner:: DistributedPlanner ;
52
47
use crate :: scheduler_server:: event_loop:: SchedulerServerEvent ;
48
+ use crate :: scheduler_server:: query_stage_scheduler:: QueryStageSchedulerEvent ;
53
49
use crate :: scheduler_server:: SchedulerServer ;
54
50
55
51
#[ tonic:: async_trait]
@@ -422,9 +418,13 @@ impl<T: 'static + AsLogicalPlan, U: 'static + AsExecutionPlan> SchedulerGrpc
422
418
tonic:: Status :: internal ( format ! ( "Could not save job metadata: {}" , e) )
423
419
} ) ?;
424
420
425
- // Create job details for the plan, like stages, tasks, etc
426
- // TODO To achieve more throughput, maybe change it to be event-based processing in the future
427
- match create_job ( self , job_id. clone ( ) , plan) . await {
421
+ match self
422
+ . post_event ( QueryStageSchedulerEvent :: JobSubmitted (
423
+ job_id. clone ( ) ,
424
+ Box :: new ( plan) ,
425
+ ) )
426
+ . await
427
+ {
428
428
Err ( error) => {
429
429
let msg = format ! ( "Job {} failed due to {}" , job_id, error) ;
430
430
warn ! ( "{}" , msg) ;
@@ -470,112 +470,6 @@ fn generate_job_id() -> String {
470
470
. collect ( )
471
471
}
472
472
473
- async fn create_job < T : ' static + AsLogicalPlan , U : ' static + AsExecutionPlan > (
474
- scheduler_server : & SchedulerServer < T , U > ,
475
- job_id : String ,
476
- plan : LogicalPlan ,
477
- ) -> Result < ( ) , BallistaError > {
478
- // create physical plan using DataFusion
479
- let plan = async move {
480
- let start = Instant :: now ( ) ;
481
-
482
- let ctx = scheduler_server. ctx . read ( ) . await . clone ( ) ;
483
- let optimized_plan = ctx. optimize ( & plan) . map_err ( |e| {
484
- let msg = format ! ( "Could not create optimized logical plan: {}" , e) ;
485
- error ! ( "{}" , msg) ;
486
- BallistaError :: General ( msg)
487
- } ) ?;
488
-
489
- debug ! ( "Calculated optimized plan: {:?}" , optimized_plan) ;
490
-
491
- let plan = ctx
492
- . create_physical_plan ( & optimized_plan)
493
- . await
494
- . map_err ( |e| {
495
- let msg = format ! ( "Could not create physical plan: {}" , e) ;
496
- error ! ( "{}" , msg) ;
497
- BallistaError :: General ( msg)
498
- } ) ;
499
-
500
- info ! (
501
- "DataFusion created physical plan in {} milliseconds" ,
502
- start. elapsed( ) . as_millis( )
503
- ) ;
504
-
505
- plan
506
- }
507
- . await ?;
508
-
509
- scheduler_server
510
- . state
511
- . save_job_metadata (
512
- & job_id,
513
- & JobStatus {
514
- status : Some ( job_status:: Status :: Running ( RunningJob { } ) ) ,
515
- } ,
516
- )
517
- . await
518
- . map_err ( |e| {
519
- warn ! ( "Could not update job {} status to running: {}" , job_id, e) ;
520
- e
521
- } ) ?;
522
-
523
- // create distributed physical plan using Ballista
524
- let mut planner = DistributedPlanner :: new ( ) ;
525
- let stages = planner
526
- . plan_query_stages ( & job_id, plan)
527
- . await
528
- . map_err ( |e| {
529
- let msg = format ! ( "Could not plan query stages: {}" , e) ;
530
- error ! ( "{}" , msg) ;
531
- BallistaError :: General ( msg)
532
- } ) ?;
533
-
534
- // save stages into state
535
- for shuffle_writer in stages {
536
- scheduler_server
537
- . state
538
- . save_stage_plan ( & job_id, shuffle_writer. stage_id ( ) , shuffle_writer. clone ( ) )
539
- . await
540
- . map_err ( |e| {
541
- let msg = format ! ( "Could not save stage plan: {}" , e) ;
542
- error ! ( "{}" , msg) ;
543
- BallistaError :: General ( msg)
544
- } ) ?;
545
- let num_partitions = shuffle_writer. output_partitioning ( ) . partition_count ( ) ;
546
- for partition_id in 0 ..num_partitions {
547
- let pending_status = TaskStatus {
548
- task_id : Some ( PartitionId {
549
- job_id : job_id. clone ( ) ,
550
- stage_id : shuffle_writer. stage_id ( ) as u32 ,
551
- partition_id : partition_id as u32 ,
552
- } ) ,
553
- status : None ,
554
- } ;
555
- scheduler_server
556
- . state
557
- . save_task_status ( & pending_status)
558
- . await
559
- . map_err ( |e| {
560
- let msg = format ! ( "Could not save task status: {}" , e) ;
561
- error ! ( "{}" , msg) ;
562
- BallistaError :: General ( msg)
563
- } ) ?;
564
- }
565
- }
566
-
567
- if let Some ( event_loop) = scheduler_server. event_loop . as_ref ( ) {
568
- // Send job_id to the scheduler channel
569
- event_loop
570
- . get_sender ( ) ?
571
- . post_event ( SchedulerServerEvent :: JobSubmitted ( job_id) )
572
- . await
573
- . unwrap ( ) ;
574
- } ;
575
-
576
- Ok ( ( ) )
577
- }
578
-
579
473
#[ cfg( all( test, feature = "sled" ) ) ]
580
474
mod test {
581
475
use std:: sync:: Arc ;
0 commit comments