3232namespace ray {
3333namespace raylet {
3434
35+ namespace {
36+ void ReplyCancelled (const std::shared_ptr<internal::Work> &work,
37+ rpc::RequestWorkerLeaseReply::SchedulingFailureType failure_type,
38+ const std::string &scheduling_failure_message) {
39+ auto reply = work->reply_ ;
40+ reply->set_canceled (true );
41+ reply->set_failure_type (failure_type);
42+ reply->set_scheduling_failure_message (scheduling_failure_message);
43+ work->send_reply_callback_ (Status::OK (), nullptr , nullptr );
44+ }
45+ } // namespace
46+
3547LocalLeaseManager::LocalLeaseManager (
3648 const NodeID &self_node_id,
3749 ClusterResourceScheduler &cluster_resource_scheduler,
@@ -410,10 +422,10 @@ void LocalLeaseManager::GrantScheduledLeasesToWorkers() {
410422 << front_lease.DebugString ();
411423 auto leases_to_grant_queue_iter = leases_to_grant_queue.begin ();
412424 while (leases_to_grant_queue_iter != leases_to_grant_queue.end ()) {
413- CancelLeaseToGrant (
414- *leases_to_grant_queue_iter,
415- rpc::RequestWorkerLeaseReply::SCHEDULING_CANCELLED_UNSCHEDULABLE,
416- " Lease granting failed due to the lease becoming infeasible." );
425+ CancelLeaseToGrantWithoutReply (*leases_to_grant_queue_iter);
426+ ReplyCancelled ( *leases_to_grant_queue_iter,
427+ rpc::RequestWorkerLeaseReply::SCHEDULING_CANCELLED_UNSCHEDULABLE,
428+ " Lease granting failed due to the lease becoming infeasible." );
417429 leases_to_grant_queue_iter =
418430 leases_to_grant_queue.erase (leases_to_grant_queue_iter);
419431 }
@@ -536,7 +548,7 @@ bool LocalLeaseManager::PoppedWorkerHandler(
536548 const rpc::Address &owner_address,
537549 const std::string &runtime_env_setup_error_message) {
538550 const auto &reply = work->reply_ ;
539- const auto &callback = work->callback_ ;
551+ const auto &send_reply_callback = work->send_reply_callback_ ;
540552 const bool canceled = work->GetState () == internal::WorkStatus::CANCELLED;
541553 const auto &lease = work->lease_ ;
542554 bool granted = false ;
@@ -650,7 +662,12 @@ bool LocalLeaseManager::PoppedWorkerHandler(
650662 RAY_LOG (DEBUG) << " Granting lease " << lease_id << " to worker "
651663 << worker->WorkerId ();
652664
653- Grant (worker, leased_workers_, work->allocated_instances_ , lease, reply, callback);
665+ Grant (worker,
666+ leased_workers_,
667+ work->allocated_instances_ ,
668+ lease,
669+ reply,
670+ send_reply_callback);
654671 erase_from_leases_to_grant_queue_fn (work, scheduling_class);
655672 granted = true ;
656673 }
@@ -660,11 +677,11 @@ bool LocalLeaseManager::PoppedWorkerHandler(
660677
661678void LocalLeaseManager::Spillback (const NodeID &spillback_to,
662679 const std::shared_ptr<internal::Work> &work) {
663- auto send_reply_callback = work->callback_ ;
680+ auto send_reply_callback = work->send_reply_callback_ ;
664681
665682 if (work->grant_or_reject_ ) {
666683 work->reply_ ->set_rejected (true );
667- send_reply_callback ();
684+ send_reply_callback (Status::OK (), nullptr , nullptr );
668685 return ;
669686 }
670687
@@ -690,7 +707,7 @@ void LocalLeaseManager::Spillback(const NodeID &spillback_to,
690707 reply->mutable_retry_at_raylet_address ()->set_port (node_info_ptr->node_manager_port ());
691708 reply->mutable_retry_at_raylet_address ()->set_node_id (spillback_to.Binary ());
692709
693- send_reply_callback ();
710+ send_reply_callback (Status::OK (), nullptr , nullptr );
694711}
695712
696713void LocalLeaseManager::LeasesUnblocked (const std::vector<LeaseID> &ready_ids) {
@@ -843,61 +860,52 @@ void LocalLeaseManager::ReleaseLeaseArgs(const LeaseID &lease_id) {
843860 }
844861}
845862
846- namespace {
847- void ReplyCancelled (const std::shared_ptr<internal::Work> &work,
848- rpc::RequestWorkerLeaseReply::SchedulingFailureType failure_type,
849- const std::string &scheduling_failure_message) {
850- auto reply = work->reply_ ;
851- auto callback = work->callback_ ;
852- reply->set_canceled (true );
853- reply->set_failure_type (failure_type);
854- reply->set_scheduling_failure_message (scheduling_failure_message);
855- callback ();
856- }
857- } // namespace
858-
859- bool LocalLeaseManager::CancelLeases (
860- std::function<bool (const std::shared_ptr<internal::Work> &)> predicate,
861- rpc::RequestWorkerLeaseReply::SchedulingFailureType failure_type,
862- const std::string &scheduling_failure_message) {
863- bool tasks_cancelled = false ;
863+ std::vector<std::shared_ptr<internal::Work>> LocalLeaseManager::CancelLeasesWithoutReply (
864+ std::function<bool (const std::shared_ptr<internal::Work> &)> predicate) {
865+ std::vector<std::shared_ptr<internal::Work>> cancelled_works;
864866
865867 ray::erase_if<SchedulingClass, std::shared_ptr<internal::Work>>(
866868 leases_to_grant_, [&](const std::shared_ptr<internal::Work> &work) {
867869 if (!predicate (work)) {
868870 return false ;
869871 }
870- CancelLeaseToGrant (work, failure_type, scheduling_failure_message );
871- tasks_cancelled = true ;
872+ CancelLeaseToGrantWithoutReply (work);
873+ cancelled_works. push_back (work) ;
872874 return true ;
873875 });
874876
875877 ray::erase_if<std::shared_ptr<internal::Work>>(
876878 waiting_lease_queue_, [&](const std::shared_ptr<internal::Work> &work) {
877- if (predicate (work)) {
878- ReplyCancelled (work, failure_type, scheduling_failure_message);
879- if (!work->lease_ .GetLeaseSpecification ().GetDependencies ().empty ()) {
880- lease_dependency_manager_.RemoveLeaseDependencies (
881- work->lease_ .GetLeaseSpecification ().LeaseId ());
882- }
883- waiting_leases_index_.erase (work->lease_ .GetLeaseSpecification ().LeaseId ());
884- tasks_cancelled = true ;
885- return true ;
886- } else {
879+ if (!predicate (work)) {
887880 return false ;
888881 }
882+ if (!work->lease_ .GetLeaseSpecification ().GetDependencies ().empty ()) {
883+ lease_dependency_manager_.RemoveLeaseDependencies (
884+ work->lease_ .GetLeaseSpecification ().LeaseId ());
885+ }
886+ waiting_leases_index_.erase (work->lease_ .GetLeaseSpecification ().LeaseId ());
887+ cancelled_works.push_back (work);
888+ return true ;
889889 });
890890
891- return tasks_cancelled ;
891+ return cancelled_works ;
892892}
893893
894- void LocalLeaseManager::CancelLeaseToGrant (
895- const std::shared_ptr<internal::Work> &work ,
894+ bool LocalLeaseManager::CancelLeases (
895+ std::function< bool ( const std::shared_ptr<internal::Work> &)> predicate ,
896896 rpc::RequestWorkerLeaseReply::SchedulingFailureType failure_type,
897897 const std::string &scheduling_failure_message) {
898+ auto cancelled_works = CancelLeasesWithoutReply (predicate);
899+ for (const auto &work : cancelled_works) {
900+ ReplyCancelled (work, failure_type, scheduling_failure_message);
901+ }
902+ return !cancelled_works.empty ();
903+ }
904+
905+ void LocalLeaseManager::CancelLeaseToGrantWithoutReply (
906+ const std::shared_ptr<internal::Work> &work) {
898907 const LeaseID lease_id = work->lease_ .GetLeaseSpecification ().LeaseId ();
899908 RAY_LOG (DEBUG) << " Canceling lease " << lease_id << " from leases_to_grant_queue." ;
900- ReplyCancelled (work, failure_type, scheduling_failure_message);
901909 if (work->GetState () == internal::WorkStatus::WAITING_FOR_WORKER) {
902910 // We've already acquired resources so we need to release them.
903911 cluster_resource_scheduler_.GetLocalResourceManager ().ReleaseWorkerResources (
@@ -962,7 +970,7 @@ void LocalLeaseManager::Grant(
962970 const std::shared_ptr<TaskResourceInstances> &allocated_instances,
963971 const RayLease &lease,
964972 rpc::RequestWorkerLeaseReply *reply,
965- std::function< void ( void )> send_reply_callback) {
973+ rpc::SendReplyCallback send_reply_callback) {
966974 const auto &lease_spec = lease.GetLeaseSpecification ();
967975
968976 if (lease_spec.IsActorCreationTask ()) {
@@ -1011,7 +1019,7 @@ void LocalLeaseManager::Grant(
10111019 }
10121020 }
10131021 // Send the result back.
1014- send_reply_callback ();
1022+ send_reply_callback (Status::OK (), nullptr , nullptr );
10151023}
10161024
10171025void LocalLeaseManager::ClearWorkerBacklog (const WorkerID &worker_id) {
0 commit comments