retryful_writer_v2.cpp 11 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385
  1. #include "retryful_writer_v2.h"
  2. #include <yt/cpp/mapreduce/client/retry_heavy_write_request.h>
  3. #include <yt/cpp/mapreduce/client/transaction.h>
  4. #include <yt/cpp/mapreduce/client/transaction_pinger.h>
  5. #include <yt/cpp/mapreduce/common/fwd.h>
  6. #include <yt/cpp/mapreduce/common/helpers.h>
  7. #include <yt/cpp/mapreduce/common/retry_lib.h>
  8. #include <yt/cpp/mapreduce/common/wait_proxy.h>
  9. #include <yt/cpp/mapreduce/http/context.h>
  10. #include <yt/cpp/mapreduce/http/helpers.h>
  11. #include <yt/cpp/mapreduce/http/http.h>
  12. #include <util/system/condvar.h>
  13. #include <queue>
  14. namespace NYT::NPrivate {
  15. ////////////////////////////////////////////////////////////////////////////////
  16. class TRetryfulWriterV2::TSentBuffer
  17. {
  18. public:
  19. TSentBuffer() = default;
  20. TSentBuffer(const TSentBuffer& ) = delete;
  21. std::pair<std::shared_ptr<std::string>, ssize_t> Snapshot() const
  22. {
  23. return {Buffer_, Size_};
  24. }
  25. void Clear()
  26. {
  27. Size_ = 0;
  28. }
  29. ssize_t Size() const
  30. {
  31. return Size_;
  32. }
  33. void Append(const void* data, ssize_t size)
  34. {
  35. auto newSize = Size_ + size;
  36. if (newSize < Capacity_) {
  37. memcpy(Buffer_->data() + Size_, data, size);
  38. } else {
  39. // Closest power of 2 exceeding new size
  40. auto newCapacity = 1 << (MostSignificantBit(newSize) + 1);
  41. newCapacity = Max<ssize_t>(64, newCapacity);
  42. auto newBuffer = std::make_shared<std::string>();
  43. newBuffer->resize(newCapacity);
  44. memcpy(newBuffer->data(), Buffer_->data(), Size_);
  45. memcpy(newBuffer->data() + Size_, data, size);
  46. Buffer_ = newBuffer;
  47. Capacity_ = newCapacity;
  48. }
  49. Size_ = newSize;
  50. }
  51. private:
  52. std::shared_ptr<std::string> Buffer_ = nullptr;
  53. ssize_t Size_ = 0;
  54. ssize_t Capacity_ = 0;
  55. };
  56. ////////////////////////////////////////////////////////////////////////////////
  57. class TRetryfulWriterV2::TSender
  58. {
  59. public:
  60. TSender(TRichYPath path, THeavyRequestRetrier::TParameters parameters)
  61. : SenderThread_(
  62. [this, path=std::move(path), parameters=std::move(parameters)] {
  63. ThreadMain(std::move(path), parameters);
  64. })
  65. {
  66. SenderThread_.SetCurrentThreadName("retryful-writer-v2-sender");
  67. SenderThread_.Start();
  68. }
  69. ~TSender()
  70. {
  71. Abort();
  72. SenderThread_.Join();
  73. }
  74. void Abort()
  75. {
  76. auto g = Guard(Lock_);
  77. SetFinishedState(EState::Aborted, g);
  78. }
  79. void Finish()
  80. {
  81. {
  82. auto g = Guard(Lock_);
  83. SetFinishedState(EState::Completed, g);
  84. }
  85. SenderThread_.Join();
  86. CheckNoError();
  87. Y_ABORT_UNLESS(TaskIdQueue_.empty());
  88. Y_ABORT_UNLESS(TaskMap_.empty());
  89. }
  90. // Return future that is complete once upload of this buffer is successfully complete
  91. std::pair<NThreading::TFuture<void>, int> StartBlock()
  92. {
  93. auto g = Guard(Lock_);
  94. CheckNoError();
  95. auto taskId = NextTaskId_++;
  96. const auto& [it, inserted] = TaskMap_.emplace(taskId, TWriteTask{});
  97. Y_ABORT_UNLESS(inserted);
  98. TaskIdQueue_.push(taskId);
  99. HaveMoreData_.Signal();
  100. it->second.SendingComplete = NThreading::NewPromise();
  101. return {it->second.SendingComplete, taskId};
  102. }
  103. void UpdateBlock(int taskId, const TSentBuffer& buffer, bool complete)
  104. {
  105. auto snapshot = buffer.Snapshot();
  106. {
  107. auto g = Guard(Lock_);
  108. CheckNoError();
  109. auto it = TaskMap_.find(taskId);
  110. Y_ABORT_UNLESS(it != TaskMap_.end());
  111. auto& writeTask = it->second;
  112. writeTask.Data = std::move(snapshot.first);
  113. writeTask.Size = snapshot.second;
  114. writeTask.BufferComplete = complete;
  115. if (TaskIdQueue_.empty() || TaskIdQueue_.back() != taskId) {
  116. TaskIdQueue_.push(taskId);
  117. }
  118. HaveMoreData_.Signal();
  119. }
  120. }
  121. private:
  122. enum class EState;
  123. private:
  124. void CheckNoError()
  125. {
  126. if (Error_) {
  127. std::rethrow_exception(Error_);
  128. }
  129. }
  130. void SetFinishedState(EState state, TGuard<TMutex>&)
  131. {
  132. if (State_ == EState::Running) {
  133. State_ = state;
  134. }
  135. HaveMoreData_.Signal();
  136. }
  137. void ThreadMain(TRichYPath path, const THeavyRequestRetrier::TParameters& parameters)
  138. {
  139. THolder<THeavyRequestRetrier> retrier;
  140. auto firstRequestParameters = parameters;
  141. auto restRequestParameters = parameters;
  142. {
  143. TNode firstPath = PathToNode(path);
  144. firstRequestParameters.Header.MergeParameters(TNode()("path", firstPath), /*overwrite*/ true);
  145. TNode restPath = PathToNode(TRichYPath(path.Path_).Append(true));
  146. restRequestParameters.Header.MergeParameters(TNode()("path", restPath), /*overwrite*/ true);
  147. }
  148. const auto* currentParameters = &firstRequestParameters;
  149. while (true) {
  150. int taskId = 0;
  151. TWriteTask task;
  152. {
  153. auto g = Guard(Lock_);
  154. while (State_ == EState::Running && TaskIdQueue_.empty()) {
  155. HaveMoreData_.Wait(Lock_);
  156. }
  157. if (
  158. State_ == EState::Aborted ||
  159. State_ == EState::Completed && TaskIdQueue_.empty()
  160. ) {
  161. break;
  162. }
  163. taskId = TaskIdQueue_.front();
  164. TaskIdQueue_.pop();
  165. if (auto it = TaskMap_.find(taskId); it != TaskMap_.end()) {
  166. task = it->second;
  167. } else {
  168. Y_ABORT();
  169. }
  170. }
  171. try {
  172. if (!retrier) {
  173. retrier = MakeHolder<THeavyRequestRetrier>(*currentParameters);
  174. }
  175. retrier->Update([task=task] {
  176. return MakeHolder<TMemoryInput>(task.Data->data(), task.Size);
  177. });
  178. if (task.BufferComplete) {
  179. retrier->Finish();
  180. retrier.Reset();
  181. }
  182. } catch (const std::exception& ex) {
  183. task.SendingComplete.SetException(std::current_exception());
  184. auto g = Guard(Lock_);
  185. Error_ = std::current_exception();
  186. return;
  187. }
  188. if (task.BufferComplete) {
  189. retrier.Reset();
  190. task.SendingComplete.SetValue();
  191. currentParameters = &restRequestParameters;
  192. auto g = Guard(Lock_);
  193. auto erased = TaskMap_.erase(taskId);
  194. Y_ABORT_UNLESS(erased == 1);
  195. }
  196. }
  197. if (State_ == EState::Completed) {
  198. auto g = Guard(Lock_);
  199. Y_ABORT_UNLESS(TaskIdQueue_.empty());
  200. Y_ABORT_UNLESS(TaskMap_.empty());
  201. }
  202. }
  203. private:
  204. struct TWriteTask
  205. {
  206. NThreading::TPromise<void> SendingComplete;
  207. std::shared_ptr<std::string> Data;
  208. ssize_t Size = 0;
  209. bool BufferComplete = false;
  210. };
  211. TMutex Lock_;
  212. TCondVar HaveMoreData_;
  213. TThread SenderThread_;
  214. THashMap<int, TWriteTask> TaskMap_;
  215. std::queue<int> TaskIdQueue_;
  216. std::exception_ptr Error_;
  217. enum class EState {
  218. Running,
  219. Completed,
  220. Aborted,
  221. };
  222. std::atomic<EState> State_ = EState::Running;
  223. int NextTaskId_ = 0;
  224. };
  225. ////////////////////////////////////////////////////////////////////////////////
  226. struct TRetryfulWriterV2::TSendTask
  227. {
  228. TSentBuffer Buffer;
  229. NThreading::TFuture<void> SentFuture = NThreading::MakeFuture();
  230. int TaskId = 0;
  231. };
  232. ////////////////////////////////////////////////////////////////////////////////
  233. TRetryfulWriterV2::TRetryfulWriterV2(
  234. IClientRetryPolicyPtr clientRetryPolicy,
  235. ITransactionPingerPtr transactionPinger,
  236. const TClientContext& context,
  237. const TTransactionId& parentId,
  238. const TString& command,
  239. const TMaybe<TFormat>& format,
  240. const TRichYPath& path,
  241. const TNode& serializedWriterOptions,
  242. ssize_t bufferSize,
  243. bool createTransaction)
  244. : BufferSize_(bufferSize)
  245. , Current_(MakeHolder<TSendTask>())
  246. , Previous_(MakeHolder<TSendTask>())
  247. {
  248. THttpHeader httpHeader("PUT", command);
  249. httpHeader.SetInputFormat(format);
  250. httpHeader.MergeParameters(serializedWriterOptions);
  251. if (createTransaction) {
  252. WriteTransaction_ = MakeHolder<TPingableTransaction>(
  253. clientRetryPolicy,
  254. context,
  255. parentId,
  256. transactionPinger->GetChildTxPinger(),
  257. TStartTransactionOptions()
  258. );
  259. auto append = path.Append_.GetOrElse(false);
  260. auto lockMode = (append ? LM_SHARED : LM_EXCLUSIVE);
  261. NDetail::NRawClient::Lock(
  262. clientRetryPolicy->CreatePolicyForGenericRequest(),
  263. context,
  264. WriteTransaction_->GetId(),
  265. path.Path_,
  266. lockMode
  267. );
  268. }
  269. THeavyRequestRetrier::TParameters parameters = {
  270. .ClientRetryPolicy = clientRetryPolicy,
  271. .TransactionPinger = transactionPinger,
  272. .Context = context,
  273. .TransactionId = WriteTransaction_ ? WriteTransaction_->GetId() : parentId,
  274. .Header = std::move(httpHeader),
  275. };
  276. Sender_ = MakeHolder<TSender>(path, parameters);
  277. DoStartBatch();
  278. }
  279. void TRetryfulWriterV2::Abort()
  280. {
  281. if (Sender_) {
  282. Sender_->Abort();
  283. }
  284. }
  285. void TRetryfulWriterV2::DoFinish()
  286. {
  287. if (Sender_) {
  288. Sender_->UpdateBlock(Current_->TaskId, Current_->Buffer, true);
  289. Sender_->Finish();
  290. Sender_.Reset();
  291. }
  292. if (WriteTransaction_) {
  293. WriteTransaction_->Commit();
  294. }
  295. }
  296. void TRetryfulWriterV2::DoStartBatch()
  297. {
  298. Previous_->SentFuture.Wait();
  299. std::swap(Previous_, Current_);
  300. auto&& [future, taskId] = Sender_->StartBlock();
  301. Current_->SentFuture = future;
  302. Current_->TaskId = taskId;
  303. Current_->Buffer.Clear();
  304. NextSizeToSend_ = SendStep_;
  305. }
  306. void TRetryfulWriterV2::DoWrite(const void* buf, size_t len)
  307. {
  308. Current_->Buffer.Append(buf, len);
  309. auto currentSize = Current_->Buffer.Size();
  310. if (currentSize >= NextSizeToSend_) {
  311. Sender_->UpdateBlock(Current_->TaskId, Current_->Buffer, false);
  312. NextSizeToSend_ = currentSize + SendStep_;
  313. }
  314. }
  315. void TRetryfulWriterV2::NotifyRowEnd()
  316. {
  317. if (Current_->Buffer.Size() >= BufferSize_) {
  318. Sender_->UpdateBlock(Current_->TaskId, Current_->Buffer, true);
  319. DoStartBatch();
  320. }
  321. }
  322. ////////////////////////////////////////////////////////////////////////////////
  323. } // namespace NYT::NPrivate