retryful_writer_v2.cpp 10 KB

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