communicator.cc 32.8 KB
Newer Older
Q
Qiao Longfei 已提交
1 2 3 4 5 6 7 8 9 10 11 12 13 14 15
/* Copyright (c) 2019 PaddlePaddle Authors. All Rights Reserved.

Licensed under the Apache License, Version 2.0 (the "License");
you may not use this file except in compliance with the License.
You may obtain a copy of the License at

    http://www.apache.org/licenses/LICENSE-2.0

Unless required by applicable law or agreed to in writing, software
distributed under the License is distributed on an "AS IS" BASIS,
WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
See the License for the specific language governing permissions and
limitations under the License. */

#include "paddle/fluid/operators/distributed/communicator.h"
16

17
#include <paddle/fluid/framework/program_desc.h>
18

19
#include <algorithm>
Q
Qiao Longfei 已提交
20
#include <chrono>  // NOLINT
21
#include <map>
Q
Qiao Longfei 已提交
22
#include <thread>  // NOLINT
23
#include <unordered_set>
24

25
#include "gflags/gflags.h"
Q
Qiao Longfei 已提交
26
#include "paddle/fluid/framework/eigen.h"
Q
Qiao Longfei 已提交
27 28
#include "paddle/fluid/framework/selected_rows.h"
#include "paddle/fluid/framework/tensor_util.h"
29
#include "paddle/fluid/framework/threadpool.h"
Q
Qiao Longfei 已提交
30
#include "paddle/fluid/framework/variable_helper.h"
C
Chengmo 已提交
31
#include "paddle/fluid/operators/distributed/distributed.h"
Q
Qiao Longfei 已提交
32 33
#include "paddle/fluid/operators/distributed/parameter_recv.h"
#include "paddle/fluid/operators/distributed/parameter_send.h"
T
tangwei12 已提交
34
#include "paddle/fluid/string/printf.h"
35
#include "paddle/fluid/string/split.h"
Q
Qiao Longfei 已提交
36

Q
Qiao Longfei 已提交
37 38 39 40
namespace paddle {
namespace operators {
namespace distributed {

41 42 43 44
using Tree =
    std::map<std::string, std::map<std::string, std::vector<std::string>>>;
using RpcCtxMap = operators::distributed::RpcCtxMap;

Q
Qiao Longfei 已提交
45 46 47 48 49 50
inline double GetCurrentUS() {
  struct timeval time;
  gettimeofday(&time, NULL);
  return 1e+6 * time.tv_sec + time.tv_usec;
}

51
Communicator::Communicator() {}
1
123malin 已提交
52

T
tangwei12 已提交
53
std::once_flag Communicator::init_flag_;
54
std::shared_ptr<Communicator> Communicator::communicator_(nullptr);
Q
can run  
Qiao Longfei 已提交
55

T
tangwei12 已提交
56 57 58 59 60 61 62
void AsyncCommunicator::InitImpl(const RpcCtxMap &send_varname_to_ctx,
                                 const RpcCtxMap &recv_varname_to_ctx,
                                 Scope *recv_scope) {
  send_varname_to_ctx_ = std::move(send_varname_to_ctx);
  recv_varname_to_ctx_ = std::move(recv_varname_to_ctx);
  recv_scope_ = std::move(recv_scope);

63 64 65 66 67
  if (send_varname_to_ctx.size() == 0) {
    VLOG(0) << "nothing need to be send, will not start send_thread";
  } else {
    send_scope_.reset(new Scope());
    for (auto &iter : send_varname_to_ctx_) {
68
      if (iter.first == STEP_COUNTER && !need_global_step_) continue;
69 70
      send_varname_to_queue_[iter.first] =
          std::make_shared<BlockingQueue<std::shared_ptr<Variable>>>(
71
              send_queue_size_);
72
    }
73
    send_threadpool_.reset(new ::ThreadPool(thread_pool_size_));
74 75 76 77 78
  }

  if (recv_varname_to_ctx.size() == 0) {
    VLOG(0) << "nothing need to be received, will not start recv_thread";
  } else {
79
    recv_threadpool_.reset(new ::ThreadPool(thread_pool_size_));
Q
Qiao Longfei 已提交
80
  }
T
tangwei12 已提交
81 82

  InitParams();
Q
Qiao Longfei 已提交
83 84
}

T
tangwei12 已提交
85 86
void AsyncCommunicator::InitParams() { RecvNoBarrier(); }

87 88 89 90 91 92 93 94
AsyncCommunicator::~AsyncCommunicator() {
  running_ = false;
  if (main_thread_) main_thread_->join();
}

void AsyncCommunicator::SendGlobalStep(int batches) {
  if (!need_global_step_) {
    return;
T
tangwei12 已提交
95 96
  }

97 98
  if (batches == 0) {
    return;
T
tangwei12 已提交
99 100
  }

101 102 103 104 105
  auto &var_name = STEP_COUNTER;
  auto *out_var = send_scope_->Var(var_name);
  auto *out_t = out_var->GetMutable<framework::LoDTensor>();
  auto *data = out_t->mutable_data<int64_t>({1}, platform::CPUPlace());
  data[0] = static_cast<int64_t>(batches);
T
tangwei12 已提交
106

107 108 109
  auto &ctx = send_varname_to_ctx_.at(var_name);
  auto send_functor = distributed::ParameterSend<float>();
  send_functor(ctx, *send_scope_, true, 1);
Q
Qiao Longfei 已提交
110 111
}

112
void AsyncCommunicator::SendByCommunicator() {
113 114 115
  std::vector<std::future<void>> task_futures;
  task_futures.reserve(send_varname_to_ctx_.size());
  VLOG(3) << "run send graph";
116

117 118 119 120 121
  auto before_run_send_graph = GetCurrentUS();
  for (auto &iter : send_varname_to_queue_) {
    auto &var_name = iter.first;
    auto &var_queue = iter.second;

122 123 124 125 126 127 128 129 130 131 132 133 134 135 136 137 138 139 140 141 142 143 144
    auto send_task = [this, &var_name, &var_queue] {
      VLOG(3) << var_name << " merge and send; ";
      std::vector<std::shared_ptr<Variable>> vars;

      int merged_var_num = 0;
      int wait_times = 0;
      while (merged_var_num < max_merge_var_num_) {
        if (var_queue->Size() == 0) {
          VLOG(4) << "wait_times -> " << wait_times;
          if (wait_times >= send_wait_times_) {
            break;
          }
          std::this_thread::sleep_for(std::chrono::milliseconds(10));
          wait_times++;
          continue;
        } else {
          wait_times = 0;

          vars.push_back(var_queue->Pop());
          merged_var_num++;
        }
      }
      auto before_merge = GetCurrentUS();
145
      if (var_name == STEP_COUNTER) {
146 147 148 149
        SendGlobalStep(merged_var_num);
        auto after_merge = GetCurrentUS();
        VLOG(3) << "merge and send " << merged_var_num << " " << var_name
                << " use time " << after_merge - before_merge;
150
        return;
Q
Qiao Longfei 已提交
151
      }
152

153 154 155 156 157 158 159 160 161 162 163 164
      auto &ctx = send_varname_to_ctx_.at(var_name);

      MergeVars<float>(var_name, vars, send_scope_.get(), ctx.merge_add);
      auto after_merge = GetCurrentUS();
      VLOG(3) << "merge " << merged_var_num << " " << var_name << " use time "
              << after_merge - before_merge;

      auto send_functor = distributed::ParameterSend<float>();
      send_functor(ctx, *send_scope_, true, 1);
      auto after_send = GetCurrentUS();
      VLOG(3) << "send " << var_name << " use time "
              << after_send - after_merge;
1
123malin 已提交
165 166 167 168 169 170 171 172 173 174 175 176

      if (var_name.rfind("@GRAD") != var_name.size() - 5) return;

      auto recv_param = var_name.substr(0, var_name.size() - 5);
      if (recv_varname_to_ctx_.find(recv_param) == recv_varname_to_ctx_.end())
        return;

      auto recv_functor = distributed::ParameterRecv<float>();
      recv_functor(recv_varname_to_ctx_.at(recv_param), *recv_scope_);
      auto after_recv = GetCurrentUS();
      VLOG(3) << "recv " << recv_param << " use time "
              << after_recv - after_send;
177 178 179 180 181 182 183 184 185 186 187 188 189 190 191 192 193 194 195 196 197 198 199 200 201 202 203 204
    };
    task_futures.emplace_back(send_threadpool_->enqueue(std::move(send_task)));
  }
  for (auto &task_f : task_futures) {
    task_f.wait();
  }
  auto after_run_send_graph = GetCurrentUS();

  VLOG(3) << "run send graph use time "
          << (after_run_send_graph - before_run_send_graph);
}

void HalfAsyncCommunicator::SendByCommunicator() {
  std::vector<std::future<void>> task_futures;
  task_futures.reserve(send_varname_to_ctx_.size());
  VLOG(3) << "run send graph";

  int batches = BatchesCounter();
  if (batches <= 0) return;

  auto before_run_send_graph = GetCurrentUS();
  for (auto &iter : send_varname_to_queue_) {
    auto &var_name = iter.first;
    auto &var_queue = iter.second;

    auto send_task = [this, batches, &var_name, &var_queue] {
      VLOG(3) << var_name << " merge and send; ";
      auto before_task = GetCurrentUS();
205 206
      std::vector<std::shared_ptr<Variable>> vars;
      vars.reserve(batches);
Q
Qiao Longfei 已提交
207

208 209 210 211
      for (int i = 0; i < batches; ++i) {
        vars.push_back(var_queue->Pop());
      }

212 213 214 215 216 217 218 219
      if (var_name == STEP_COUNTER) {
        SendGlobalStep(batches);
        auto end_task = GetCurrentUS();
        VLOG(3) << "merge " << batches << " " << var_name << " use time "
                << end_task - before_task;
        return;
      }

220 221 222 223 224 225 226 227 228 229 230 231
      auto &ctx = send_varname_to_ctx_.at(var_name);

      auto before_merge = GetCurrentUS();
      MergeVars<float>(var_name, vars, send_scope_.get(), ctx.merge_add);
      auto after_merge = GetCurrentUS();
      VLOG(3) << "merge " << batches << " " << var_name << " use time "
              << after_merge - before_merge;

      auto send_functor = distributed::ParameterSend<float>();
      send_functor(ctx, *send_scope_, true, 1);
      auto after_send = GetCurrentUS();
      VLOG(3) << "send " << var_name << " use time "
232 233 234 235 236 237 238 239 240 241 242 243 244 245
              << after_send - before_task;

      if (var_name.rfind("@GRAD") != var_name.size() - 5) return;

      auto recv_param = var_name.substr(0, var_name.size() - 5);
      if (recv_varname_to_ctx_.find(recv_param) == recv_varname_to_ctx_.end())
        return;

      auto recv_functor = distributed::ParameterRecv<float>();
      recv_functor(recv_varname_to_ctx_.at(recv_param), *recv_scope_);
      auto after_recv = GetCurrentUS();
      VLOG(3) << "recv " << recv_param << " use time "
              << after_recv - after_send;
      return;
246 247
    };
    task_futures.emplace_back(send_threadpool_->enqueue(std::move(send_task)));
Q
Qiao Longfei 已提交
248
  }
249 250 251 252 253 254
  for (auto &task_f : task_futures) {
    task_f.wait();
  }
  auto after_run_send_graph = GetCurrentUS();

  VLOG(3) << "run send graph use time "
255
          << (after_run_send_graph - before_run_send_graph);
Q
Qiao Longfei 已提交
256 257
}

258 259 260 261 262 263
void AsyncCommunicator::MainThread() {
  VLOG(3) << "MainThread start and wait";

  while (waiting_ && running_) {
    std::this_thread::sleep_for(std::chrono::milliseconds(100));
    VLOG(3) << "wait for running";
264 265
  }

266
  while (running_) {
267 268 269 270 271 272 273 274 275 276 277 278
    SendByCommunicator();
    BarrierSend();
  }
  VLOG(3) << "communicator stopped, send thread exit";
}

void HalfAsyncCommunicator::MainThread() {
  VLOG(3) << "MainThread start and wait";

  while (waiting_ && running_) {
    std::this_thread::sleep_for(std::chrono::milliseconds(100));
    VLOG(3) << "wait for running";
279
  }
280 281 282 283 284 285 286 287 288

  while (running_) {
    SendByCommunicator();
    BarrierSend();
    RecvByCommunicator();
    BarrierRecv();
    BarrierWeakUp();
  }
  VLOG(3) << "communicator stopped, send thread exit";
289 290
}

291
void AsyncCommunicator::RecvByCommunicator() {
T
tangwei12 已提交
292 293
  VLOG(3) << "parallel run recv graph";
  if (!running_) return;
294 295 296 297 298
  RecvNoBarrier();
  VLOG(3) << "run recv graph use time";
}

void AsyncCommunicator::RecvNoBarrier() {
T
tangwei12 已提交
299 300
  std::vector<std::future<void>> task_futures;
  task_futures.reserve(recv_varname_to_ctx_.size());
301

T
tangwei12 已提交
302 303
  for (auto &iter : recv_varname_to_ctx_) {
    auto recv_task = [this, &iter] {
304
      auto before_task = GetCurrentUS();
T
tangwei12 已提交
305 306
      auto &var_name = iter.first;
      auto recv_functor = distributed::ParameterRecv<float>();
T
tangwei12 已提交
307
      recv_functor(iter.second, *recv_scope_);
308 309 310
      auto end_task = GetCurrentUS();
      VLOG(1) << "recv var " << var_name << " use time "
              << (end_task - before_task);
T
tangwei12 已提交
311 312 313
    };
    task_futures.emplace_back(recv_threadpool_->enqueue(std::move(recv_task)));
  }
314

T
tangwei12 已提交
315 316 317
  for (auto &task : task_futures) {
    task.wait();
  }
318 319
}

T
tangwei12 已提交
320
void AsyncCommunicator::Start() {
321
  VLOG(3) << "Communicator start";
322 323 324
  if (!communicator_) {
    VLOG(0) << "Communicator is not inited, do nothing";
  } else {
325
    VLOG(3) << "start send thread and recv thread";
326
    waiting_ = true;
327
    running_ = true;
328
    BarrierTriggerReset(max_merge_var_num_);
329
    // start send and recv thread
330 331
    main_thread_.reset(
        new std::thread(std::bind(&AsyncCommunicator::MainThread, this)));
332 333 334
  }
}

T
tangwei12 已提交
335
void AsyncCommunicator::Stop() {
336
  VLOG(3) << "Communicator stop";
337 338 339 340
  running_ = false;
  if (!communicator_) {
    VLOG(0) << "Communicator is not inited, do nothing";
  } else {
341
    if (main_thread_) {
342
      VLOG(3) << "stop send thread";
343 344
      main_thread_->join();
      main_thread_.reset(nullptr);
345
    }
Q
Qiao Longfei 已提交
346
  }
347
  VLOG(3) << "Communicator stop done";
Q
Qiao Longfei 已提交
348 349
}

350 351 352
void AsyncCommunicator::Send(const std::vector<std::string> &var_names,
                             const std::vector<std::string> &var_tables,
                             const framework::Scope &scope) {
353 354
  waiting_ = false;

355
  PADDLE_ENFORCE_EQ(
356 357 358 359
      var_tables.size(), 1,
      platform::errors::InvalidArgument("var_tables.size() == 1 is permitted"));

  auto table_name = var_tables[0];
360 361 362 363

  if (table_name == STEP_COUNTER && !need_global_step_) return;

  auto before_send_op = GetCurrentUS();
364 365 366 367 368 369 370 371 372 373 374 375 376 377 378 379 380 381 382 383 384 385 386 387 388 389 390 391 392 393 394 395 396 397
  auto &queue = send_varname_to_queue_.at(table_name);

  if (table_name == STEP_COUNTER) {
    auto tmp_var = std::make_shared<Variable>();
    auto *tensor = tmp_var->GetMutable<framework::LoDTensor>();
    tensor->Resize(framework::make_ddim({1}));
    auto *out_d = tensor->mutable_data<int64_t>(platform::CPUPlace());
    out_d[0] = 1;
    queue->Push(tmp_var);
  } else {
    PADDLE_ENFORCE_GE(var_names.size(), 1,
                      platform::errors::InvalidArgument(
                          "var_names.size() >= 1 is permitted"));

    auto *var = scope.FindVar(var_names[0]);

    PADDLE_ENFORCE_EQ(
        var->IsInitialized(), true,
        platform::errors::InvalidArgument("grad var should be inited"));

    auto tmp_var = std::make_shared<Variable>();
    if (var->IsType<framework::SelectedRows>()) {
      framework::CopyVariable(*var, tmp_var.get());
      queue->Push(tmp_var);
    } else if (var->IsType<framework::LoDTensor>()) {
      // push var into send queue by var_name
      auto var_name = var_names[0];
      framework::CopyVariable(*var, tmp_var.get());
      queue->Push(tmp_var);
    } else {
      PADDLE_THROW(platform::errors::InvalidArgument(
          "unknown var type to copy, only support LoDTensor/SelectedRows"));
    }
  }
398 399 400
  auto after_send_op = GetCurrentUS();
  VLOG(3) << "send to " << table_name << " with queue size " << queue->Size()
          << ", use time " << (after_send_op - before_send_op);
401
}
402

403 404 405 406
void HalfAsyncCommunicator::Clean() {
  for (auto &iter : send_varname_to_queue_) {
    auto &var_name = iter.first;
    auto &var_queue = iter.second;
407

408 409
    while (var_queue->Size() > 0) {
      var_queue->Pop();
410 411
    }

412 413 414 415
    VLOG(3) << "clean var: " << var_name << " done";
  }
}

T
tangwei12 已提交
416
int HalfAsyncCommunicator::BatchesCounter() {
417 418 419 420 421 422
  while (running_) {
    if (barrier_counter_.load() >= barrier_trigger_.load() &&
        barrier_trigger_.load() != 0) {
      break;
    } else {
      std::this_thread::sleep_for(std::chrono::milliseconds(10));
423
    }
424
  }
425

426 427
  return barrier_counter_.load();
}
C
Chengmo 已提交
428

429 430
void HalfAsyncCommunicator::Barrier() {
  barrier_counter_++;
431

432 433 434
  if (!running_) {
    VLOG(3) << "Communicator is not running, release barrier";
    return;
435 436
  }

437 438 439
  {
    std::unique_lock<std::mutex> lk(barrier_mutex_);
    barrier_cond_.wait(lk, [this] { return (barrier_counter_ == 0); });
440
  }
441
}
442

443 444 445 446
void HalfAsyncCommunicator::BarrierTriggerDecrement() {
  barrier_trigger_--;
  VLOG(3) << "BarrierTriggerDecrement decrement barrier trigger to "
          << barrier_trigger_.load();
447 448
}

449 450 451 452 453
void HalfAsyncCommunicator::BarrierTriggerReset(int initial_val) {
  barrier_trigger_.store(initial_val);

  VLOG(3) << "BarrierTriggerReset reset barrier trigger to "
          << barrier_trigger_.load();
454 455
}

456 457 458 459 460 461 462 463 464 465 466 467 468 469 470
void HalfAsyncCommunicator::BarrierWeakUp() {
  barrier_counter_.store(0);
  barrier_cond_.notify_all();
}

void SyncCommunicator::BarrierSend() {
  if (!running_) return;

  distributed::RPCClient *rpc_client =
      distributed::RPCClient::GetInstance<RPCCLIENT_T>(trainer_id_);

  std::vector<distributed::VarHandlePtr> rets;

  for (auto &ep : pserver_endpoints_) {
    rets.push_back(rpc_client->AsyncSendBatchBarrier(ep));
471
  }
472 473 474 475 476 477 478

  for (size_t i = 0; i < rets.size(); i++) {
    PADDLE_ENFORCE_NE(rets[i]->Wait(), 0U, platform::errors::External(
                                               "internal error in RPCClient"));
  }

  VLOG(4) << "BarrierSend with SyncCommunicator";
479 480
}

481 482 483 484 485 486 487 488 489
void SyncCommunicator::BarrierRecv() {
  if (!running_) return;

  distributed::RPCClient *rpc_client =
      distributed::RPCClient::GetInstance<RPCCLIENT_T>(trainer_id_);

  std::vector<distributed::VarHandlePtr> rets;
  for (auto &ep : pserver_endpoints_) {
    rets.push_back(rpc_client->AsyncSendFetchBarrier(ep));
490 491
  }

492 493 494
  for (size_t i = 0; i < rets.size(); i++) {
    PADDLE_ENFORCE_NE(rets[i]->Wait(), 0U, platform::errors::External(
                                               "internal error in RPCClient"));
495
  }
496 497

  VLOG(4) << "BarrierRecv with SyncCommunicator";
498 499
}

500 501 502 503 504 505
void GeoCommunicator::InitImpl(const RpcCtxMap &send_varname_to_ctx,
                               const RpcCtxMap &recv_varname_to_ctx,
                               Scope *recv_scope) {
  send_varname_to_ctx_ = std::move(send_varname_to_ctx);
  recv_varname_to_ctx_ = std::move(recv_varname_to_ctx);
  recv_scope_ = std::move(recv_scope);
506

507 508 509 510 511 512 513 514 515 516 517 518 519 520 521
  PADDLE_ENFORCE_GT(
      send_varname_to_ctx.size(), 0,
      platform::errors::InvalidArgument("send var contexts can not be zero"));

  send_scope_.reset(new Scope());
  for (auto &iter : send_varname_to_ctx_) {
    auto &varname = iter.first;

    if (varname == STEP_COUNTER) {
      send_varname_to_queue_[varname] =
          std::make_shared<BlockingQueue<std::shared_ptr<Variable>>>(
              send_queue_size_);
    } else {
      auto &send_ctx = iter.second;

522
      send_var_nums_ += send_ctx.splited_varnames.size();
523
      if (!send_ctx.is_sparse) {
C
Chengmo 已提交
524
        continue;
525
      }
526 527 528 529 530 531 532 533 534 535
      int pserver_num = static_cast<int>(send_ctx.epmap.size());
      for (int ep_idx = 0; ep_idx < pserver_num; ep_idx++) {
        sparse_id_queues_.insert(
            std::pair<std::string, std::shared_ptr<BlockingQueue<
                                       std::shared_ptr<std::vector<int64_t>>>>>(
                send_ctx.splited_varnames[ep_idx],
                std::make_shared<
                    BlockingQueue<std::shared_ptr<std::vector<int64_t>>>>(
                    send_queue_size_)));
      }
536 537
    }
  }
538
  send_threadpool_.reset(new ::ThreadPool(thread_pool_size_));
539

540 541 542 543
  if (recv_varname_to_ctx.size() == 0) {
    VLOG(0) << "nothing need to be received, will not start recv_thread";
  } else {
    recv_threadpool_.reset(new ::ThreadPool(thread_pool_size_));
544 545
  }

546 547 548
  delta_scope_.reset(new Scope());
  old_scope_.reset(new Scope());
  pserver_scope_.reset(new Scope());
549

T
tangwei12 已提交
550
  InitParams();
551 552
}

553 554 555 556
void GeoCommunicator::Send(const std::vector<std::string> &var_names,
                           const std::vector<std::string> &var_tables,
                           const framework::Scope &scope) {
  waiting_ = false;
557 558 559
  PADDLE_ENFORCE_EQ(
      var_tables.size(), 1,
      platform::errors::InvalidArgument("var_tables.size() == 1 is permitted"));
560

561 562
  auto table_name = var_tables[0];
  if (table_name == STEP_COUNTER) return;
563

564 565 566
  auto before_send = GetCurrentUS();
  size_t splited_var_nums =
      send_varname_to_ctx_[table_name].splited_varnames.size();
567

568
  std::unordered_map<std::string, std::unordered_set<int64_t>> ids_table;
569

570 571 572 573 574 575 576 577 578 579 580 581 582
  for (size_t j = 0; j < splited_var_nums; j++) {
    ids_table.insert(std::pair<std::string, std::unordered_set<int64_t>>(
        send_varname_to_ctx_[table_name].splited_varnames[j],
        std::unordered_set<int64_t>()));
  }
  auto *var = scope.FindVar(var_names[0]);
  auto &rows = var->Get<framework::SelectedRows>().rows();

  // insert ids which has not been record
  for (size_t j = 0; j < rows.size(); j++) {
    auto ep_idx = rows[j] % splited_var_nums;
    ids_table.at(send_varname_to_ctx_[table_name].splited_varnames[ep_idx])
        .insert(rows[j]);
583
  }
584

585 586 587 588 589 590 591 592 593 594 595
  auto before_push = GetCurrentUS();
  for (auto &iter : ids_table) {
    auto &key = iter.first;
    auto &sparse_ids_set = iter.second;
    auto sparse_ids_vec = std::make_shared<std::vector<int64_t>>();
    sparse_ids_vec->assign(sparse_ids_set.begin(), sparse_ids_set.end());
    sparse_id_queues_.at(key)->Push(sparse_ids_vec);
    VLOG(3) << "push " << sparse_ids_vec->size() << " ids to " << key
            << "'s queue";
  }
  auto after_send = GetCurrentUS();
596 597
  VLOG(3) << "run send " << table_name << " op finish. using "
          << (before_push - before_send) << "; " << (after_send - before_push);
598 599
}

600 601
void GeoCommunicator::MainThread() {
  VLOG(3) << "MainThread start and wait";
602

603 604 605 606
  while (waiting_ && running_) {
    std::this_thread::sleep_for(std::chrono::milliseconds(100));
    VLOG(3) << "wait for running";
  }
C
Chengmo 已提交
607

608 609 610
  while (running_) {
    std::vector<std::future<void>> tasks;
    tasks.reserve(send_var_nums_);
C
Chengmo 已提交
611

612 613 614 615
    for (auto &iter : send_varname_to_ctx_) {
      auto &var_name = iter.first;
      auto &send_ctx = iter.second;
      int pserver_num = static_cast<int>(send_ctx.epmap.size());
616
      if (send_ctx.is_sparse) {
617 618 619 620 621 622 623 624 625 626 627 628 629 630 631 632 633 634 635 636 637 638 639 640 641 642
        for (int ep_idx = 0; ep_idx < pserver_num; ep_idx++) {
          auto send_recv_task = [this, ep_idx, &var_name] {
            auto before_send_sparse = GetCurrentUS();
            if (var_name == STEP_COUNTER) {
              return;
            }
            auto send_varname =
                send_varname_to_ctx_.at(var_name).splited_varnames[ep_idx];
            auto sparse_ids = MergeSparseIds(send_varname);
            if (sparse_ids.size() == 0) {
              return;
            }
            SendSparse(var_name, ep_idx, sparse_ids);
            auto after_send_sparse = GetCurrentUS();
            RecvSparse(var_name, ep_idx);
            auto after_recv_sparse = GetCurrentUS();
            VLOG(3)
                << "send recv "
                << send_varname_to_ctx_.at(var_name).splited_varnames[ep_idx]
                << " finish, using " << (after_send_sparse - before_send_sparse)
                << " and " << (after_recv_sparse - after_send_sparse)
                << "; total = " << (after_recv_sparse - before_send_sparse);
          };
          tasks.emplace_back(
              send_threadpool_->enqueue(std::move(send_recv_task)));
        }
643
      } else {
644 645 646 647 648 649 650 651 652
        auto send_recv_task = [this, &var_name, &send_ctx] {
          if (var_name == STEP_COUNTER) {
            return;
          }
          SendDense(var_name);
          RecvDense(var_name);
        };
        tasks.emplace_back(
            send_threadpool_->enqueue(std::move(send_recv_task)));
653
      }
654 655 656 657
    }
    for (auto &task : tasks) {
      task.wait();
    }
658 659
  }
}
C
Chengmo 已提交
660

661 662 663 664 665 666 667 668 669 670 671 672 673 674 675 676 677 678 679 680 681 682 683 684
std::vector<int64_t> GeoCommunicator::MergeSparseIds(
    const std::string &send_varname) {
  size_t merge_num = 0, wait_times = 0;
  std::unordered_set<int64_t> sparse_ids;
  while (merge_num < static_cast<size_t>(max_merge_var_num_)) {
    VLOG(3) << "Merge Number of " << send_varname << " = " << merge_num;
    if (sparse_id_queues_.at(send_varname)->Size() > 0) {
      wait_times = 0;
      std::shared_ptr<std::vector<int64_t>> pop_ids =
          sparse_id_queues_.at(send_varname)->Pop();
      for (size_t j = 0; j < pop_ids->size(); j++) {
        sparse_ids.insert(pop_ids->at(j));
      }
      merge_num += 1;
      VLOG(3) << "sparse_id_queues_(" << send_varname << ") pushed";
    } else if (sparse_id_queues_.at(send_varname)->Size() == 0) {
      VLOG(3) << "wait_times -> " << wait_times;
      if (wait_times >= static_cast<size_t>(send_wait_times_)) {
        break;
      }
      std::this_thread::sleep_for(std::chrono::milliseconds(10));
      wait_times++;
      continue;
    }
685
  }
686 687 688 689 690 691 692 693 694 695 696
  std::vector<int64_t> res;
  res.assign(sparse_ids.begin(), sparse_ids.end());
  return res;
}
void GeoCommunicator::SendSparse(const std::string &varname, int ep_idx,
                                 const std::vector<int64_t> &sparse_ids) {
  auto &rpc_ctx = send_varname_to_ctx_.at(varname);
  auto send_varname = rpc_ctx.splited_varnames[ep_idx];
  auto trainer_id = rpc_ctx.trainer_id;
  auto endpoint = rpc_ctx.epmap[ep_idx];
  auto pserver_num = rpc_ctx.epmap.size();
697

698 699 700 701 702 703 704 705
  auto *var_latest = recv_scope_->FindVar(varname);

  PADDLE_ENFORCE_EQ(var_latest->IsInitialized(), true,
                    platform::errors::Unavailable(
                        "%s is not initialized, please check", varname));
  auto &t_latest = var_latest->Get<framework::LoDTensor>();

  auto dims1 = t_latest.dims()[1];
C
Chengmo 已提交
706 707

  auto cpu_ctx = paddle::platform::CPUDeviceContext();
708
  auto *var_delta = delta_scope_->Var(send_varname);
709
  auto *t_delta = var_delta->GetMutable<framework::SelectedRows>();
710

711 712
  auto *t_value = t_delta->mutable_value();
  t_value->mutable_data<float>(
713
      framework::make_ddim({static_cast<int64_t>(sparse_ids.size()), dims1}),
714
      cpu_ctx.GetPlace());
C
Chengmo 已提交
715

716 717
  std::vector<std::vector<std::vector<float> *>> values;
  auto *ins = distributed::LargeScaleKV::GetInstance();
718
  ins->Get(varname)->Get(sparse_ids, {"Param"}, &values);
C
Chengmo 已提交
719

720 721
  auto blas = math::GetBlas<platform::CPUDeviceContext, float>(cpu_ctx);
  float coefficient = 1.0 / static_cast<float>(trainers_);
C
Chengmo 已提交
722

723 724
  for (auto j = 0; j < static_cast<int>(sparse_ids.size()); ++j) {
    blas.VSUB(dims1, t_latest.data<float>() + sparse_ids[j] * dims1,
725 726 727 728
              values[j][0]->data(), t_value->data<float>() + j * dims1);
    blas.SCAL(dims1, coefficient, t_value->data<float>() + j * dims1);
    blas.VADD(dims1, values[j][0]->data(), t_value->data<float>() + j * dims1,
              values[j][0]->data());
C
Chengmo 已提交
729 730
  }

731 732 733 734 735 736 737 738 739 740 741 742 743 744 745 746
  std::vector<int64_t> send_rows;
  send_rows.reserve(sparse_ids.size());
  for (auto idx : sparse_ids) {
    send_rows.push_back(idx / pserver_num);
  }
  t_delta->set_height(rpc_ctx.height_sections[ep_idx]);
  t_delta->set_rows(send_rows);

  platform::DeviceContextPool &pool = platform::DeviceContextPool::Instance();
  auto &cpu_ctx_send = *pool.Get(platform::CPUPlace());
  distributed::RPCClient *rpc_client =
      distributed::RPCClient::GetInstance<RPCCLIENT_T>(trainer_id);

  auto ret = rpc_client->AsyncSendVar(endpoint, cpu_ctx_send,
                                      *delta_scope_.get(), send_varname);
  ret->Wait();
747 748
}

749 750 751
void GeoCommunicator::SendDense(const std::string &varname) {
  auto *var_latest = recv_scope_->FindVar(varname);
  auto *var_timestamp = old_scope_->FindVar(varname);
752

753 754 755 756 757 758
  PADDLE_ENFORCE_EQ(var_latest->IsInitialized(), true,
                    platform::errors::Unavailable(
                        "%s is not initialized, please check", varname));
  PADDLE_ENFORCE_EQ(var_timestamp->IsInitialized(), true,
                    platform::errors::Unavailable(
                        "%s is not initialized, please check", varname));
759

760 761
  auto &t_latest = var_latest->Get<framework::LoDTensor>();
  auto t_timestamp = var_timestamp->GetMutable<framework::LoDTensor>();
C
Chengmo 已提交
762

763 764 765 766
  auto cpu_ctx = paddle::platform::CPUDeviceContext();
  auto *var_delta = delta_scope_->Var(varname);
  auto *t_delta = var_delta->GetMutable<framework::LoDTensor>();
  t_delta->mutable_data<float>(t_latest.dims(), cpu_ctx.GetPlace());
C
Chengmo 已提交
767

768 769 770
  auto blas = math::GetBlas<platform::CPUDeviceContext, float>(cpu_ctx);
  blas.VSUB(t_latest.numel(), t_latest.data<float>(),
            t_timestamp->data<float>(), t_delta->data<float>());
C
Chengmo 已提交
771

772 773
  float coefficient = 1.0 / static_cast<float>(trainers_);
  blas.SCAL(t_latest.numel(), coefficient, t_delta->data<float>());
C
Chengmo 已提交
774

775 776
  blas.VADD(t_latest.numel(), t_timestamp->data<float>(),
            t_delta->data<float>(), t_timestamp->data<float>());
777

778 779 780 781
  auto &ctx = send_varname_to_ctx_.at(varname);
  auto send = distributed::ParameterSend<float>();
  send(ctx, *delta_scope_, true, 1);
}
782

783
void GeoCommunicator::RecvByCommunicator() { return; }
784

785 786 787 788 789 790
void GeoCommunicator::RecvSparse(const std::string &varname, int ep_idx) {
  auto train_id = recv_varname_to_ctx_.at(varname).trainer_id;
  auto endpoint = recv_varname_to_ctx_.at(varname).epmap[ep_idx];
  auto splited_var_name =
      recv_varname_to_ctx_.at(varname).splited_varnames[ep_idx];
  auto pserver_num = recv_varname_to_ctx_.at(varname).epmap.size();
791

792 793 794 795
  platform::DeviceContextPool &pool = platform::DeviceContextPool::Instance();
  auto &cpu_ctx_recv = *pool.Get(platform::CPUPlace());
  distributed::RPCClient *rpc_client =
      distributed::RPCClient::GetInstance<RPCCLIENT_T>(train_id);
796

797 798 799 800 801
  auto *var_psrever = pserver_scope_->Var(splited_var_name);
  auto handle = rpc_client->AsyncGetVar(endpoint, cpu_ctx_recv,
                                        *pserver_scope_.get(), splited_var_name,
                                        splited_var_name, splited_var_name);
  handle->Wait();
802

803
  auto *var_latest = recv_scope_->FindVar(varname);
804

805 806 807 808
  PADDLE_ENFORCE_EQ(
      var_psrever->IsInitialized(), true,
      platform::errors::Unavailable(
          "%s in pserver scope is not initialized, please check", varname));
809

810 811 812
  std::vector<int64_t> ids;
  ids.assign(var_psrever->Get<framework::SelectedRows>().rows().begin(),
             var_psrever->Get<framework::SelectedRows>().rows().end());
813

814 815 816 817 818
  for (size_t j = 0; j < ids.size(); j++) {
    ids[j] = ids[j] * pserver_num + ep_idx;
  }

  VLOG(3) << "RecvSparse receive var: " << splited_var_name
819
          << " ids Size: " << ids.size();
820

821
  auto t_psrever = var_psrever->Get<framework::SelectedRows>().value();
822

823
  std::vector<std::vector<std::vector<float> *>> old_values;
824

825 826
  auto *ins = distributed::LargeScaleKV::GetInstance();
  ins->Get(varname)->Get(ids, {"Param"}, &old_values);
827

828
  auto *t_latest = var_latest->GetMutable<framework::LoDTensor>();
829

830 831
  auto dims1 = t_latest->dims()[1];
  auto numel = ids.size() * dims1;
832

833 834 835 836 837 838 839 840 841 842 843 844 845 846
  std::vector<float> v_delta;
  v_delta.resize(numel);

  auto cpu_ctx = paddle::platform::CPUDeviceContext();
  auto blas = math::GetBlas<platform::CPUDeviceContext, float>(cpu_ctx);

  for (auto j = 0; j < static_cast<int>(ids.size()); ++j) {
    blas.VSUB(dims1, t_psrever.data<float>() + j * dims1,
              old_values[j][0]->data(), v_delta.data() + j * dims1);
    blas.VADD(dims1, t_latest->data<float>() + ids[j] * dims1,
              v_delta.data() + j * dims1,
              t_latest->data<float>() + ids[j] * dims1);
    blas.VCOPY(dims1, t_psrever.data<float>() + j * dims1,
               old_values[j][0]->data());
847 848 849
  }
}

850 851 852 853
void GeoCommunicator::RecvDense(const std::string &varname) {
  auto *var_latest = recv_scope_->FindVar(varname);
  auto *var_timestamp = old_scope_->FindVar(varname);
  auto *var_psrever = pserver_scope_->Var(varname);
854

855 856
  auto &ctx = recv_varname_to_ctx_.at(varname);
  auto recv = distributed::ParameterRecv<float>();
T
tangwei12 已提交
857
  recv(ctx, *pserver_scope_);
858

859 860 861 862
  PADDLE_ENFORCE_EQ(
      var_psrever->IsInitialized(), true,
      platform::errors::Unavailable(
          "%s in pserver scope is not initialized, please check", varname));
863

864 865 866
  auto t_psrever = var_psrever->Get<framework::LoDTensor>();
  auto t_latest = var_latest->GetMutable<framework::LoDTensor>();
  auto t_timestamp = var_timestamp->GetMutable<framework::LoDTensor>();
867

868 869 870 871
  auto cpu_ctx = paddle::platform::CPUDeviceContext();
  auto *var_delta = delta_scope_->Var(varname);
  auto *t_delta = var_delta->GetMutable<framework::LoDTensor>();
  t_delta->mutable_data<float>(t_latest->dims(), cpu_ctx.GetPlace());
872

873 874 875 876 877 878 879
  auto blas = math::GetBlas<platform::CPUDeviceContext, float>(cpu_ctx);
  blas.VSUB(t_latest->numel(), t_psrever.data<float>(),
            t_timestamp->data<float>(), t_delta->data<float>());
  blas.VADD(t_latest->numel(), t_latest->data<float>(), t_delta->data<float>(),
            t_latest->data<float>());
  blas.VCOPY(t_latest->numel(), t_psrever.data<float>(),
             t_timestamp->data<float>());
880 881
}

T
tangwei12 已提交
882
void GeoCommunicator::InitParams() {
883 884
  std::vector<std::future<void>> tasks;
  tasks.reserve(recv_varname_to_ctx_.size());
885

886 887 888
  for (auto &iter : recv_varname_to_ctx_) {
    auto &var_name = iter.first;
    auto &recv_ctx = iter.second;
889

890 891 892 893 894 895
    auto recv_task = [this, &var_name, &recv_ctx] {
      if (!recv_ctx.is_sparse) {
        InitDense(var_name);
      }
    };
    tasks.emplace_back(send_threadpool_->enqueue(std::move(recv_task)));
896 897
  }

898 899
  for (auto &task : tasks) {
    task.wait();
900
  }
901
  InitSparse();
902
}
903

904 905 906
void GeoCommunicator::InitDense(const std::string varname) {
  auto &ctx = recv_varname_to_ctx_.at(varname);
  auto recv = distributed::ParameterRecv<float>();
T
tangwei12 已提交
907 908 909 910 911 912 913 914 915
  recv(ctx, *recv_scope_);

  auto *global_var = recv_scope_->FindVar(varname);
  global_var->GetMutable<framework::LoDTensor>();

  auto *old_var = old_scope_->Var(varname);
  old_var->GetMutable<framework::LoDTensor>();

  framework::CopyVariable(*global_var, old_var);
916 917
  VLOG(1) << "init dense variable " << varname << " done";
}
T
tangwei12 已提交
918

919 920
void GeoCommunicator::InitSparse() {
  auto sparse_metas = string::split_string<std::string>(sparse_attrs_, "#");
T
tangwei12 已提交
921

922 923
  std::vector<distributed::SparseMeta> metas;
  std::vector<int64_t> dicts;
T
tangwei12 已提交
924

925 926 927 928 929 930 931 932 933 934 935 936 937 938 939 940 941 942
  for (auto &sparse_meta : sparse_metas) {
    auto attrs = string::split_string<std::string>(sparse_meta, ":");

    auto meta = distributed::SparseMeta();
    meta.name = attrs[0];
    meta.value_names = {"Param"};

    auto dic = string::split_string<std::string>(attrs[1], ",");
    dicts.push_back(std::stoi(dic[0]));
    meta.value_dims = {std::stoi(dic[1])};
    meta.mode = distributed::Mode::training;
    meta.grad_name = "none";
    meta.cached_varnames = {};
    meta.initializer_attrs = string::split_string<std::string>(attrs[2]);
    meta.entry = "none";

    VLOG(3) << "add sparse meta: " << meta.ToString();
    metas.push_back(meta);
T
tangwei12 已提交
943 944
  }

945
  LargeScaleKV::Init(metas);
T
tangwei12 已提交
946

T
tangwei12 已提交
947 948 949
  for (auto &meta : metas) {
    auto &ctx = recv_varname_to_ctx_.at(meta.name);
    auto recv = distributed::ParameterRecv<float>();
T
tangwei12 已提交
950

T
tangwei12 已提交
951 952 953 954
    auto *global_var = recv_scope_->FindVar(meta.name);
    auto global_value = global_var->Get<framework::LoDTensor>();
    auto rows = global_value.dims()[0];
    auto dim1 = global_value.dims()[1];
T
tangwei12 已提交
955

T
tangwei12 已提交
956 957 958 959 960 961 962 963 964 965 966 967
    recv(ctx, *recv_scope_);
    VLOG(1) << "recv " << meta.name << " with global scope for init";

    auto n_rows = global_var->Get<framework::LoDTensor>().dims()[0];

    PADDLE_ENFORCE_EQ(
        rows, n_rows,
        platform::errors::InvalidArgument(
            "global var: %s origin dim must equal recved rows", meta.name));

    std::vector<int64_t> ids(rows);
    std::iota(ids.begin(), ids.end(), 0);
T
tangwei12 已提交
968

969
    auto *ins = distributed::LargeScaleKV::GetInstance();
T
tangwei12 已提交
970 971 972 973
    std::vector<std::vector<std::vector<float> *>> values;

    ins->Get(meta.name)->Init(ids);
    ins->Get(meta.name)->Get(ids, {"Param"}, &values);
974

T
tangwei12 已提交
975 976 977 978 979 980 981
    auto blas = math::GetBlas<platform::CPUDeviceContext, float>(
        paddle::platform::CPUDeviceContext());

    for (auto &id : ids) {
      blas.VCOPY(dim1, global_value.data<float>() + id * dim1,
                 values[id][0]->data());
    }
T
tangwei12 已提交
982 983
  }

984
  VLOG(3) << "init sparse variable done";
T
tangwei12 已提交
985 986
}

Q
Qiao Longfei 已提交
987 988 989
}  // namespace distributed
}  // namespace operators
}  // namespace paddle