column_family_test.cc 114 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406407408409410411412413414415416417418419420421422423424425426427428429430431432433434435436437438439440441442443444445446447448449450451452453454455456457458459460461462463464465466467468469470471472473474475476477478479480481482483484485486487488489490491492493494495496497498499500501502503504505506507508509510511512513514515516517518519520521522523524525526527528529530531532533534535536537538539540541542543544545546547548549550551552553554555556557558559560561562563564565566567568569570571572573574575576577578579580581582583584585586587588589590591592593594595596597598599600601602603604605606607608609610611612613614615616617618619620621622623624625626627628629630631632633634635636637638639640641642643644645646647648649650651652653654655656657658659660661662663664665666667668669670671672673674675676677678679680681682683684685686687688689690691692693694695696697698699700701702703704705706707708709710711712713714715716717718719720721722723724725726727728729730731732733734735736737738739740741742743744745746747748749750751752753754755756757758759760761762763764765766767768769770771772773774775776777778779780781782783784785786787788789790791792793794795796797798799800801802803804805806807808809810811812813814815816817818819820821822823824825826827828829830831832833834835836837838839840841842843844845846847848849850851852853854855856857858859860861862863864865866867868869870871872873874875876877878879880881882883884885886887888889890891892893894895896897898899900901902903904905906907908909910911912913914915916917918919920921922923924925926927928929930931932933934935936937938939940941942943944945946947948949950951952953954955956957958959960961962963964965966967968969970971972973974975976977978979980981982983984985986987988989990991992993994995996997998999100010011002100310041005100610071008100910101011101210131014101510161017101810191020102110221023102410251026102710281029103010311032103310341035103610371038103910401041104210431044104510461047104810491050105110521053105410551056105710581059106010611062106310641065106610671068106910701071107210731074107510761077107810791080108110821083108410851086108710881089109010911092109310941095109610971098109911001101110211031104110511061107110811091110111111121113111411151116111711181119112011211122112311241125112611271128112911301131113211331134113511361137113811391140114111421143114411451146114711481149115011511152115311541155115611571158115911601161116211631164116511661167116811691170117111721173117411751176117711781179118011811182118311841185118611871188118911901191119211931194119511961197119811991200120112021203120412051206120712081209121012111212121312141215121612171218121912201221122212231224122512261227122812291230123112321233123412351236123712381239124012411242124312441245124612471248124912501251125212531254125512561257125812591260126112621263126412651266126712681269127012711272127312741275127612771278127912801281128212831284128512861287128812891290129112921293129412951296129712981299130013011302130313041305130613071308130913101311131213131314131513161317131813191320132113221323132413251326132713281329133013311332133313341335133613371338133913401341134213431344134513461347134813491350135113521353135413551356135713581359136013611362136313641365136613671368136913701371137213731374137513761377137813791380138113821383138413851386138713881389139013911392139313941395139613971398139914001401140214031404140514061407140814091410141114121413141414151416141714181419142014211422142314241425142614271428142914301431143214331434143514361437143814391440144114421443144414451446144714481449145014511452145314541455145614571458145914601461146214631464146514661467146814691470147114721473147414751476147714781479148014811482148314841485148614871488148914901491149214931494149514961497149814991500150115021503150415051506150715081509151015111512151315141515151615171518151915201521152215231524152515261527152815291530153115321533153415351536153715381539154015411542154315441545154615471548154915501551155215531554155515561557155815591560156115621563156415651566156715681569157015711572157315741575157615771578157915801581158215831584158515861587158815891590159115921593159415951596159715981599160016011602160316041605160616071608160916101611161216131614161516161617161816191620162116221623162416251626162716281629163016311632163316341635163616371638163916401641164216431644164516461647164816491650165116521653165416551656165716581659166016611662166316641665166616671668166916701671167216731674167516761677167816791680168116821683168416851686168716881689169016911692169316941695169616971698169917001701170217031704170517061707170817091710171117121713171417151716171717181719172017211722172317241725172617271728172917301731173217331734173517361737173817391740174117421743174417451746174717481749175017511752175317541755175617571758175917601761176217631764176517661767176817691770177117721773177417751776177717781779178017811782178317841785178617871788178917901791179217931794179517961797179817991800180118021803180418051806180718081809181018111812181318141815181618171818181918201821182218231824182518261827182818291830183118321833183418351836183718381839184018411842184318441845184618471848184918501851185218531854185518561857185818591860186118621863186418651866186718681869187018711872187318741875187618771878187918801881188218831884188518861887188818891890189118921893189418951896189718981899190019011902190319041905190619071908190919101911191219131914191519161917191819191920192119221923192419251926192719281929193019311932193319341935193619371938193919401941194219431944194519461947194819491950195119521953195419551956195719581959196019611962196319641965196619671968196919701971197219731974197519761977197819791980198119821983198419851986198719881989199019911992199319941995199619971998199920002001200220032004200520062007200820092010201120122013201420152016201720182019202020212022202320242025202620272028202920302031203220332034203520362037203820392040204120422043204420452046204720482049205020512052205320542055205620572058205920602061206220632064206520662067206820692070207120722073207420752076207720782079208020812082208320842085208620872088208920902091209220932094209520962097209820992100210121022103210421052106210721082109211021112112211321142115211621172118211921202121212221232124212521262127212821292130213121322133213421352136213721382139214021412142214321442145214621472148214921502151215221532154215521562157215821592160216121622163216421652166216721682169217021712172217321742175217621772178217921802181218221832184218521862187218821892190219121922193219421952196219721982199220022012202220322042205220622072208220922102211221222132214221522162217221822192220222122222223222422252226222722282229223022312232223322342235223622372238223922402241224222432244224522462247224822492250225122522253225422552256225722582259226022612262226322642265226622672268226922702271227222732274227522762277227822792280228122822283228422852286228722882289229022912292229322942295229622972298229923002301230223032304230523062307230823092310231123122313231423152316231723182319232023212322232323242325232623272328232923302331233223332334233523362337233823392340234123422343234423452346234723482349235023512352235323542355235623572358235923602361236223632364236523662367236823692370237123722373237423752376237723782379238023812382238323842385238623872388238923902391239223932394239523962397239823992400240124022403240424052406240724082409241024112412241324142415241624172418241924202421242224232424242524262427242824292430243124322433243424352436243724382439244024412442244324442445244624472448244924502451245224532454245524562457245824592460246124622463246424652466246724682469247024712472247324742475247624772478247924802481248224832484248524862487248824892490249124922493249424952496249724982499250025012502250325042505250625072508250925102511251225132514251525162517251825192520252125222523252425252526252725282529253025312532253325342535253625372538253925402541254225432544254525462547254825492550255125522553255425552556255725582559256025612562256325642565256625672568256925702571257225732574257525762577257825792580258125822583258425852586258725882589259025912592259325942595259625972598259926002601260226032604260526062607260826092610261126122613261426152616261726182619262026212622262326242625262626272628262926302631263226332634263526362637263826392640264126422643264426452646264726482649265026512652265326542655265626572658265926602661266226632664266526662667266826692670267126722673267426752676267726782679268026812682268326842685268626872688268926902691269226932694269526962697269826992700270127022703270427052706270727082709271027112712271327142715271627172718271927202721272227232724272527262727272827292730273127322733273427352736273727382739274027412742274327442745274627472748274927502751275227532754275527562757275827592760276127622763276427652766276727682769277027712772277327742775277627772778277927802781278227832784278527862787278827892790279127922793279427952796279727982799280028012802280328042805280628072808280928102811281228132814281528162817281828192820282128222823282428252826282728282829283028312832283328342835283628372838283928402841284228432844284528462847284828492850285128522853285428552856285728582859286028612862286328642865286628672868286928702871287228732874287528762877287828792880288128822883288428852886288728882889289028912892289328942895289628972898289929002901290229032904290529062907290829092910291129122913291429152916291729182919292029212922292329242925292629272928292929302931293229332934293529362937293829392940294129422943294429452946294729482949295029512952295329542955295629572958295929602961296229632964296529662967296829692970297129722973297429752976297729782979298029812982298329842985298629872988298929902991299229932994299529962997299829993000300130023003300430053006300730083009301030113012301330143015301630173018301930203021302230233024302530263027302830293030303130323033303430353036303730383039304030413042304330443045304630473048304930503051305230533054305530563057305830593060306130623063306430653066306730683069307030713072307330743075307630773078307930803081308230833084308530863087308830893090309130923093309430953096309730983099310031013102310331043105310631073108310931103111311231133114311531163117311831193120312131223123312431253126312731283129313031313132313331343135313631373138313931403141314231433144314531463147314831493150315131523153315431553156315731583159316031613162316331643165316631673168316931703171317231733174317531763177317831793180318131823183318431853186318731883189319031913192319331943195319631973198319932003201320232033204320532063207320832093210321132123213321432153216321732183219322032213222322332243225322632273228322932303231323232333234323532363237323832393240324132423243324432453246324732483249325032513252325332543255325632573258325932603261326232633264326532663267326832693270327132723273327432753276327732783279328032813282328332843285328632873288328932903291329232933294329532963297329832993300330133023303330433053306330733083309331033113312331333143315331633173318331933203321332233233324332533263327332833293330333133323333333433353336333733383339334033413342334333443345334633473348334933503351335233533354335533563357335833593360336133623363336433653366336733683369337033713372337333743375337633773378337933803381338233833384338533863387
  1. // Copyright (c) 2011-present, Facebook, Inc. All rights reserved.
  2. // This source code is licensed under both the GPLv2 (found in the
  3. // COPYING file in the root directory) and Apache 2.0 License
  4. // (found in the LICENSE.Apache file in the root directory).
  5. //
  6. // Copyright (c) 2011 The LevelDB Authors. All rights reserved.
  7. // Use of this source code is governed by a BSD-style license that can be
  8. // found in the LICENSE file. See the AUTHORS file for names of contributors.
  9. #include <algorithm>
  10. #include <vector>
  11. #include <string>
  12. #include <thread>
  13. #include "db/db_impl/db_impl.h"
  14. #include "db/db_test_util.h"
  15. #include "memtable/hash_skiplist_rep.h"
  16. #include "options/options_parser.h"
  17. #include "port/port.h"
  18. #include "port/stack_trace.h"
  19. #include "rocksdb/db.h"
  20. #include "rocksdb/env.h"
  21. #include "rocksdb/iterator.h"
  22. #include "rocksdb/utilities/object_registry.h"
  23. #include "test_util/fault_injection_test_env.h"
  24. #include "test_util/sync_point.h"
  25. #include "test_util/testharness.h"
  26. #include "test_util/testutil.h"
  27. #include "util/coding.h"
  28. #include "util/string_util.h"
  29. #include "utilities/merge_operators.h"
  30. namespace ROCKSDB_NAMESPACE {
  31. static const int kValueSize = 1000;
  32. namespace {
  33. std::string RandomString(Random* rnd, int len) {
  34. std::string r;
  35. test::RandomString(rnd, len, &r);
  36. return r;
  37. }
  38. } // anonymous namespace
  39. // counts how many operations were performed
  40. class EnvCounter : public EnvWrapper {
  41. public:
  42. explicit EnvCounter(Env* base)
  43. : EnvWrapper(base), num_new_writable_file_(0) {}
  44. int GetNumberOfNewWritableFileCalls() {
  45. return num_new_writable_file_;
  46. }
  47. Status NewWritableFile(const std::string& f, std::unique_ptr<WritableFile>* r,
  48. const EnvOptions& soptions) override {
  49. ++num_new_writable_file_;
  50. return EnvWrapper::NewWritableFile(f, r, soptions);
  51. }
  52. private:
  53. std::atomic<int> num_new_writable_file_;
  54. };
  55. class ColumnFamilyTestBase : public testing::Test {
  56. public:
  57. explicit ColumnFamilyTestBase(uint32_t format) : rnd_(139), format_(format) {
  58. Env* base_env = Env::Default();
  59. #ifndef ROCKSDB_LITE
  60. const char* test_env_uri = getenv("TEST_ENV_URI");
  61. if (test_env_uri) {
  62. Env* test_env = nullptr;
  63. Status s = Env::LoadEnv(test_env_uri, &test_env, &env_guard_);
  64. base_env = test_env;
  65. EXPECT_OK(s);
  66. EXPECT_NE(Env::Default(), base_env);
  67. }
  68. #endif // !ROCKSDB_LITE
  69. EXPECT_NE(nullptr, base_env);
  70. env_ = new EnvCounter(base_env);
  71. dbname_ = test::PerThreadDBPath("column_family_test");
  72. db_options_.create_if_missing = true;
  73. db_options_.fail_if_options_file_error = true;
  74. db_options_.env = env_;
  75. DestroyDB(dbname_, Options(db_options_, column_family_options_));
  76. }
  77. ~ColumnFamilyTestBase() override {
  78. std::vector<ColumnFamilyDescriptor> column_families;
  79. for (auto h : handles_) {
  80. ColumnFamilyDescriptor cfdescriptor;
  81. h->GetDescriptor(&cfdescriptor);
  82. column_families.push_back(cfdescriptor);
  83. }
  84. Close();
  85. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  86. Destroy(column_families);
  87. delete env_;
  88. }
  89. BlockBasedTableOptions GetBlockBasedTableOptions() {
  90. BlockBasedTableOptions options;
  91. options.format_version = format_;
  92. return options;
  93. }
  94. // Return the value to associate with the specified key
  95. Slice Value(int k, std::string* storage) {
  96. if (k == 0) {
  97. // Ugh. Random seed of 0 used to produce no entropy. This code
  98. // preserves the implementation that was in place when all of the
  99. // magic values in this file were picked.
  100. *storage = std::string(kValueSize, ' ');
  101. return Slice(*storage);
  102. } else {
  103. Random r(k);
  104. return test::RandomString(&r, kValueSize, storage);
  105. }
  106. }
  107. void Build(int base, int n, int flush_every = 0) {
  108. std::string key_space, value_space;
  109. WriteBatch batch;
  110. for (int i = 0; i < n; i++) {
  111. if (flush_every != 0 && i != 0 && i % flush_every == 0) {
  112. DBImpl* dbi = reinterpret_cast<DBImpl*>(db_);
  113. dbi->TEST_FlushMemTable();
  114. }
  115. int keyi = base + i;
  116. Slice key(DBTestBase::Key(keyi));
  117. batch.Clear();
  118. batch.Put(handles_[0], key, Value(keyi, &value_space));
  119. batch.Put(handles_[1], key, Value(keyi, &value_space));
  120. batch.Put(handles_[2], key, Value(keyi, &value_space));
  121. ASSERT_OK(db_->Write(WriteOptions(), &batch));
  122. }
  123. }
  124. void CheckMissed() {
  125. uint64_t next_expected = 0;
  126. uint64_t missed = 0;
  127. int bad_keys = 0;
  128. int bad_values = 0;
  129. int correct = 0;
  130. std::string value_space;
  131. for (int cf = 0; cf < 3; cf++) {
  132. next_expected = 0;
  133. Iterator* iter = db_->NewIterator(ReadOptions(false, true), handles_[cf]);
  134. for (iter->SeekToFirst(); iter->Valid(); iter->Next()) {
  135. uint64_t key;
  136. Slice in(iter->key());
  137. in.remove_prefix(3);
  138. if (!ConsumeDecimalNumber(&in, &key) || !in.empty() ||
  139. key < next_expected) {
  140. bad_keys++;
  141. continue;
  142. }
  143. missed += (key - next_expected);
  144. next_expected = key + 1;
  145. if (iter->value() != Value(static_cast<int>(key), &value_space)) {
  146. bad_values++;
  147. } else {
  148. correct++;
  149. }
  150. }
  151. delete iter;
  152. }
  153. ASSERT_EQ(0, bad_keys);
  154. ASSERT_EQ(0, bad_values);
  155. ASSERT_EQ(0, missed);
  156. (void)correct;
  157. }
  158. void Close() {
  159. for (auto h : handles_) {
  160. if (h) {
  161. db_->DestroyColumnFamilyHandle(h);
  162. }
  163. }
  164. handles_.clear();
  165. names_.clear();
  166. delete db_;
  167. db_ = nullptr;
  168. }
  169. Status TryOpen(std::vector<std::string> cf,
  170. std::vector<ColumnFamilyOptions> options = {}) {
  171. std::vector<ColumnFamilyDescriptor> column_families;
  172. names_.clear();
  173. for (size_t i = 0; i < cf.size(); ++i) {
  174. column_families.push_back(ColumnFamilyDescriptor(
  175. cf[i], options.size() == 0 ? column_family_options_ : options[i]));
  176. names_.push_back(cf[i]);
  177. }
  178. return DB::Open(db_options_, dbname_, column_families, &handles_, &db_);
  179. }
  180. Status OpenReadOnly(std::vector<std::string> cf,
  181. std::vector<ColumnFamilyOptions> options = {}) {
  182. std::vector<ColumnFamilyDescriptor> column_families;
  183. names_.clear();
  184. for (size_t i = 0; i < cf.size(); ++i) {
  185. column_families.push_back(ColumnFamilyDescriptor(
  186. cf[i], options.size() == 0 ? column_family_options_ : options[i]));
  187. names_.push_back(cf[i]);
  188. }
  189. return DB::OpenForReadOnly(db_options_, dbname_, column_families, &handles_,
  190. &db_);
  191. }
  192. #ifndef ROCKSDB_LITE // ReadOnlyDB is not supported
  193. void AssertOpenReadOnly(std::vector<std::string> cf,
  194. std::vector<ColumnFamilyOptions> options = {}) {
  195. ASSERT_OK(OpenReadOnly(cf, options));
  196. }
  197. #endif // !ROCKSDB_LITE
  198. void Open(std::vector<std::string> cf,
  199. std::vector<ColumnFamilyOptions> options = {}) {
  200. ASSERT_OK(TryOpen(cf, options));
  201. }
  202. void Open() {
  203. Open({"default"});
  204. }
  205. DBImpl* dbfull() { return reinterpret_cast<DBImpl*>(db_); }
  206. int GetProperty(int cf, std::string property) {
  207. std::string value;
  208. EXPECT_TRUE(dbfull()->GetProperty(handles_[cf], property, &value));
  209. #ifndef CYGWIN
  210. return std::stoi(value);
  211. #else
  212. return std::strtol(value.c_str(), 0 /* off */, 10 /* base */);
  213. #endif
  214. }
  215. bool IsDbWriteStopped() {
  216. #ifndef ROCKSDB_LITE
  217. uint64_t v;
  218. EXPECT_TRUE(dbfull()->GetIntProperty("rocksdb.is-write-stopped", &v));
  219. return (v == 1);
  220. #else
  221. return dbfull()->TEST_write_controler().IsStopped();
  222. #endif // !ROCKSDB_LITE
  223. }
  224. uint64_t GetDbDelayedWriteRate() {
  225. #ifndef ROCKSDB_LITE
  226. uint64_t v;
  227. EXPECT_TRUE(
  228. dbfull()->GetIntProperty("rocksdb.actual-delayed-write-rate", &v));
  229. return v;
  230. #else
  231. if (!dbfull()->TEST_write_controler().NeedsDelay()) {
  232. return 0;
  233. }
  234. return dbfull()->TEST_write_controler().delayed_write_rate();
  235. #endif // !ROCKSDB_LITE
  236. }
  237. void Destroy(const std::vector<ColumnFamilyDescriptor>& column_families =
  238. std::vector<ColumnFamilyDescriptor>()) {
  239. Close();
  240. ASSERT_OK(DestroyDB(dbname_, Options(db_options_, column_family_options_),
  241. column_families));
  242. }
  243. void CreateColumnFamilies(
  244. const std::vector<std::string>& cfs,
  245. const std::vector<ColumnFamilyOptions> options = {}) {
  246. int cfi = static_cast<int>(handles_.size());
  247. handles_.resize(cfi + cfs.size());
  248. names_.resize(cfi + cfs.size());
  249. for (size_t i = 0; i < cfs.size(); ++i) {
  250. const auto& current_cf_opt =
  251. options.size() == 0 ? column_family_options_ : options[i];
  252. ASSERT_OK(
  253. db_->CreateColumnFamily(current_cf_opt, cfs[i], &handles_[cfi]));
  254. names_[cfi] = cfs[i];
  255. #ifndef ROCKSDB_LITE // RocksDBLite does not support GetDescriptor
  256. // Verify the CF options of the returned CF handle.
  257. ColumnFamilyDescriptor desc;
  258. ASSERT_OK(handles_[cfi]->GetDescriptor(&desc));
  259. RocksDBOptionsParser::VerifyCFOptions(desc.options, current_cf_opt);
  260. #endif // !ROCKSDB_LITE
  261. cfi++;
  262. }
  263. }
  264. void Reopen(const std::vector<ColumnFamilyOptions> options = {}) {
  265. std::vector<std::string> names;
  266. for (auto name : names_) {
  267. if (name != "") {
  268. names.push_back(name);
  269. }
  270. }
  271. Close();
  272. assert(options.size() == 0 || names.size() == options.size());
  273. Open(names, options);
  274. }
  275. void CreateColumnFamiliesAndReopen(const std::vector<std::string>& cfs) {
  276. CreateColumnFamilies(cfs);
  277. Reopen();
  278. }
  279. void DropColumnFamilies(const std::vector<int>& cfs) {
  280. for (auto cf : cfs) {
  281. ASSERT_OK(db_->DropColumnFamily(handles_[cf]));
  282. db_->DestroyColumnFamilyHandle(handles_[cf]);
  283. handles_[cf] = nullptr;
  284. names_[cf] = "";
  285. }
  286. }
  287. void PutRandomData(int cf, int num, int key_value_size, bool save = false) {
  288. if (cf >= static_cast<int>(keys_.size())) {
  289. keys_.resize(cf + 1);
  290. }
  291. for (int i = 0; i < num; ++i) {
  292. // 10 bytes for key, rest is value
  293. if (!save) {
  294. ASSERT_OK(Put(cf, test::RandomKey(&rnd_, 11),
  295. RandomString(&rnd_, key_value_size - 10)));
  296. } else {
  297. std::string key = test::RandomKey(&rnd_, 11);
  298. keys_[cf].insert(key);
  299. ASSERT_OK(Put(cf, key, RandomString(&rnd_, key_value_size - 10)));
  300. }
  301. }
  302. db_->FlushWAL(false);
  303. }
  304. #ifndef ROCKSDB_LITE // TEST functions in DB are not supported in lite
  305. void WaitForFlush(int cf) {
  306. ASSERT_OK(dbfull()->TEST_WaitForFlushMemTable(handles_[cf]));
  307. }
  308. void WaitForCompaction() {
  309. ASSERT_OK(dbfull()->TEST_WaitForCompact());
  310. }
  311. uint64_t MaxTotalInMemoryState() {
  312. return dbfull()->TEST_MaxTotalInMemoryState();
  313. }
  314. void AssertMaxTotalInMemoryState(uint64_t value) {
  315. ASSERT_EQ(value, MaxTotalInMemoryState());
  316. }
  317. #endif // !ROCKSDB_LITE
  318. Status Put(int cf, const std::string& key, const std::string& value) {
  319. return db_->Put(WriteOptions(), handles_[cf], Slice(key), Slice(value));
  320. }
  321. Status Merge(int cf, const std::string& key, const std::string& value) {
  322. return db_->Merge(WriteOptions(), handles_[cf], Slice(key), Slice(value));
  323. }
  324. Status Flush(int cf) {
  325. return db_->Flush(FlushOptions(), handles_[cf]);
  326. }
  327. std::string Get(int cf, const std::string& key) {
  328. ReadOptions options;
  329. options.verify_checksums = true;
  330. std::string result;
  331. Status s = db_->Get(options, handles_[cf], Slice(key), &result);
  332. if (s.IsNotFound()) {
  333. result = "NOT_FOUND";
  334. } else if (!s.ok()) {
  335. result = s.ToString();
  336. }
  337. return result;
  338. }
  339. void CompactAll(int cf) {
  340. ASSERT_OK(db_->CompactRange(CompactRangeOptions(), handles_[cf], nullptr,
  341. nullptr));
  342. }
  343. void Compact(int cf, const Slice& start, const Slice& limit) {
  344. ASSERT_OK(
  345. db_->CompactRange(CompactRangeOptions(), handles_[cf], &start, &limit));
  346. }
  347. int NumTableFilesAtLevel(int level, int cf) {
  348. return GetProperty(cf,
  349. "rocksdb.num-files-at-level" + ToString(level));
  350. }
  351. #ifndef ROCKSDB_LITE
  352. // Return spread of files per level
  353. std::string FilesPerLevel(int cf) {
  354. std::string result;
  355. int last_non_zero_offset = 0;
  356. for (int level = 0; level < dbfull()->NumberLevels(handles_[cf]); level++) {
  357. int f = NumTableFilesAtLevel(level, cf);
  358. char buf[100];
  359. snprintf(buf, sizeof(buf), "%s%d", (level ? "," : ""), f);
  360. result += buf;
  361. if (f > 0) {
  362. last_non_zero_offset = static_cast<int>(result.size());
  363. }
  364. }
  365. result.resize(last_non_zero_offset);
  366. return result;
  367. }
  368. #endif
  369. void AssertFilesPerLevel(const std::string& value, int cf) {
  370. #ifndef ROCKSDB_LITE
  371. ASSERT_EQ(value, FilesPerLevel(cf));
  372. #else
  373. (void) value;
  374. (void) cf;
  375. #endif
  376. }
  377. #ifndef ROCKSDB_LITE // GetLiveFilesMetaData is not supported
  378. int CountLiveFiles() {
  379. std::vector<LiveFileMetaData> metadata;
  380. db_->GetLiveFilesMetaData(&metadata);
  381. return static_cast<int>(metadata.size());
  382. }
  383. #endif // !ROCKSDB_LITE
  384. void AssertCountLiveFiles(int expected_value) {
  385. #ifndef ROCKSDB_LITE
  386. ASSERT_EQ(expected_value, CountLiveFiles());
  387. #else
  388. (void) expected_value;
  389. #endif
  390. }
  391. // Do n memtable flushes, each of which produces an sstable
  392. // covering the range [small,large].
  393. void MakeTables(int cf, int n, const std::string& small,
  394. const std::string& large) {
  395. for (int i = 0; i < n; i++) {
  396. ASSERT_OK(Put(cf, small, "begin"));
  397. ASSERT_OK(Put(cf, large, "end"));
  398. ASSERT_OK(db_->Flush(FlushOptions(), handles_[cf]));
  399. }
  400. }
  401. #ifndef ROCKSDB_LITE // GetSortedWalFiles is not supported
  402. int CountLiveLogFiles() {
  403. int micros_wait_for_log_deletion = 20000;
  404. env_->SleepForMicroseconds(micros_wait_for_log_deletion);
  405. int ret = 0;
  406. VectorLogPtr wal_files;
  407. Status s;
  408. // GetSortedWalFiles is a flakey function -- it gets all the wal_dir
  409. // children files and then later checks for their existence. if some of the
  410. // log files doesn't exist anymore, it reports an error. it does all of this
  411. // without DB mutex held, so if a background process deletes the log file
  412. // while the function is being executed, it returns an error. We retry the
  413. // function 10 times to avoid the error failing the test
  414. for (int retries = 0; retries < 10; ++retries) {
  415. wal_files.clear();
  416. s = db_->GetSortedWalFiles(wal_files);
  417. if (s.ok()) {
  418. break;
  419. }
  420. }
  421. EXPECT_OK(s);
  422. for (const auto& wal : wal_files) {
  423. if (wal->Type() == kAliveLogFile) {
  424. ++ret;
  425. }
  426. }
  427. return ret;
  428. return 0;
  429. }
  430. #endif // !ROCKSDB_LITE
  431. void AssertCountLiveLogFiles(int value) {
  432. #ifndef ROCKSDB_LITE // GetSortedWalFiles is not supported
  433. ASSERT_EQ(value, CountLiveLogFiles());
  434. #else
  435. (void) value;
  436. #endif // !ROCKSDB_LITE
  437. }
  438. void AssertNumberOfImmutableMemtables(std::vector<int> num_per_cf) {
  439. assert(num_per_cf.size() == handles_.size());
  440. #ifndef ROCKSDB_LITE // GetProperty is not supported in lite
  441. for (size_t i = 0; i < num_per_cf.size(); ++i) {
  442. ASSERT_EQ(num_per_cf[i], GetProperty(static_cast<int>(i),
  443. "rocksdb.num-immutable-mem-table"));
  444. }
  445. #endif // !ROCKSDB_LITE
  446. }
  447. void CopyFile(const std::string& source, const std::string& destination,
  448. uint64_t size = 0) {
  449. const EnvOptions soptions;
  450. std::unique_ptr<SequentialFile> srcfile;
  451. ASSERT_OK(env_->NewSequentialFile(source, &srcfile, soptions));
  452. std::unique_ptr<WritableFile> destfile;
  453. ASSERT_OK(env_->NewWritableFile(destination, &destfile, soptions));
  454. if (size == 0) {
  455. // default argument means copy everything
  456. ASSERT_OK(env_->GetFileSize(source, &size));
  457. }
  458. char buffer[4096];
  459. Slice slice;
  460. while (size > 0) {
  461. uint64_t one = std::min(uint64_t(sizeof(buffer)), size);
  462. ASSERT_OK(srcfile->Read(one, &slice, buffer));
  463. ASSERT_OK(destfile->Append(slice));
  464. size -= slice.size();
  465. }
  466. ASSERT_OK(destfile->Close());
  467. }
  468. int GetSstFileCount(std::string path) {
  469. std::vector<std::string> files;
  470. DBTestBase::GetSstFiles(env_, path, &files);
  471. return static_cast<int>(files.size());
  472. }
  473. void RecalculateWriteStallConditions(ColumnFamilyData* cfd,
  474. const MutableCFOptions& mutable_cf_options) {
  475. // add lock to avoid race condition between
  476. // `RecalculateWriteStallConditions` which writes to CFStats and
  477. // background `DBImpl::DumpStats()` threads which read CFStats
  478. dbfull()->TEST_LockMutex();
  479. cfd->RecalculateWriteStallConditions(mutable_cf_options);
  480. dbfull()-> TEST_UnlockMutex();
  481. }
  482. std::vector<ColumnFamilyHandle*> handles_;
  483. std::vector<std::string> names_;
  484. std::vector<std::set<std::string>> keys_;
  485. ColumnFamilyOptions column_family_options_;
  486. DBOptions db_options_;
  487. std::string dbname_;
  488. DB* db_ = nullptr;
  489. EnvCounter* env_;
  490. std::shared_ptr<Env> env_guard_;
  491. Random rnd_;
  492. uint32_t format_;
  493. };
  494. class ColumnFamilyTest
  495. : public ColumnFamilyTestBase,
  496. virtual public ::testing::WithParamInterface<uint32_t> {
  497. public:
  498. ColumnFamilyTest() : ColumnFamilyTestBase(GetParam()) {}
  499. };
  500. INSTANTIATE_TEST_CASE_P(FormatDef, ColumnFamilyTest,
  501. testing::Values(test::kDefaultFormatVersion));
  502. INSTANTIATE_TEST_CASE_P(FormatLatest, ColumnFamilyTest,
  503. testing::Values(test::kLatestFormatVersion));
  504. TEST_P(ColumnFamilyTest, DontReuseColumnFamilyID) {
  505. for (int iter = 0; iter < 3; ++iter) {
  506. Open();
  507. CreateColumnFamilies({"one", "two", "three"});
  508. for (size_t i = 0; i < handles_.size(); ++i) {
  509. auto cfh = reinterpret_cast<ColumnFamilyHandleImpl*>(handles_[i]);
  510. ASSERT_EQ(i, cfh->GetID());
  511. }
  512. if (iter == 1) {
  513. Reopen();
  514. }
  515. DropColumnFamilies({3});
  516. Reopen();
  517. if (iter == 2) {
  518. // this tests if max_column_family is correctly persisted with
  519. // WriteSnapshot()
  520. Reopen();
  521. }
  522. CreateColumnFamilies({"three2"});
  523. // ID 3 that was used for dropped column family "three" should not be
  524. // reused
  525. auto cfh3 = reinterpret_cast<ColumnFamilyHandleImpl*>(handles_[3]);
  526. ASSERT_EQ(4U, cfh3->GetID());
  527. Close();
  528. Destroy();
  529. }
  530. }
  531. #ifndef ROCKSDB_LITE
  532. TEST_P(ColumnFamilyTest, CreateCFRaceWithGetAggProperty) {
  533. Open();
  534. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  535. {{"DBImpl::WriteOptionsFile:1",
  536. "ColumnFamilyTest.CreateCFRaceWithGetAggProperty:1"},
  537. {"ColumnFamilyTest.CreateCFRaceWithGetAggProperty:2",
  538. "DBImpl::WriteOptionsFile:2"}});
  539. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  540. ROCKSDB_NAMESPACE::port::Thread thread(
  541. [&] { CreateColumnFamilies({"one"}); });
  542. TEST_SYNC_POINT("ColumnFamilyTest.CreateCFRaceWithGetAggProperty:1");
  543. uint64_t pv;
  544. db_->GetAggregatedIntProperty(DB::Properties::kEstimateTableReadersMem, &pv);
  545. TEST_SYNC_POINT("ColumnFamilyTest.CreateCFRaceWithGetAggProperty:2");
  546. thread.join();
  547. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  548. }
  549. #endif // !ROCKSDB_LITE
  550. class FlushEmptyCFTestWithParam
  551. : public ColumnFamilyTestBase,
  552. virtual public testing::WithParamInterface<std::tuple<uint32_t, bool>> {
  553. public:
  554. FlushEmptyCFTestWithParam()
  555. : ColumnFamilyTestBase(std::get<0>(GetParam())),
  556. allow_2pc_(std::get<1>(GetParam())) {}
  557. // Required if inheriting from testing::WithParamInterface<>
  558. static void SetUpTestCase() {}
  559. static void TearDownTestCase() {}
  560. bool allow_2pc_;
  561. };
  562. TEST_P(FlushEmptyCFTestWithParam, FlushEmptyCFTest) {
  563. std::unique_ptr<FaultInjectionTestEnv> fault_env(
  564. new FaultInjectionTestEnv(env_));
  565. db_options_.env = fault_env.get();
  566. db_options_.allow_2pc = allow_2pc_;
  567. Open();
  568. CreateColumnFamilies({"one", "two"});
  569. // Generate log file A.
  570. ASSERT_OK(Put(1, "foo", "v1")); // seqID 1
  571. Reopen();
  572. // Log file A is not dropped after reopening because default column family's
  573. // min log number is 0.
  574. // It flushes to SST file X
  575. ASSERT_OK(Put(1, "foo", "v1")); // seqID 2
  576. ASSERT_OK(Put(1, "bar", "v2")); // seqID 3
  577. // Current log file is file B now. While flushing, a new log file C is created
  578. // and is set to current. Boths' min log number is set to file C in memory, so
  579. // after flushing file B is deleted. At the same time, the min log number of
  580. // default CF is not written to manifest. Log file A still remains.
  581. // Flushed to SST file Y.
  582. Flush(1);
  583. Flush(0);
  584. ASSERT_OK(Put(1, "bar", "v3")); // seqID 4
  585. ASSERT_OK(Put(1, "foo", "v4")); // seqID 5
  586. db_->FlushWAL(false);
  587. // Preserve file system state up to here to simulate a crash condition.
  588. fault_env->SetFilesystemActive(false);
  589. std::vector<std::string> names;
  590. for (auto name : names_) {
  591. if (name != "") {
  592. names.push_back(name);
  593. }
  594. }
  595. Close();
  596. fault_env->ResetState();
  597. // Before opening, there are four files:
  598. // Log file A contains seqID 1
  599. // Log file C contains seqID 4, 5
  600. // SST file X contains seqID 1
  601. // SST file Y contains seqID 2, 3
  602. // Min log number:
  603. // default CF: 0
  604. // CF one, two: C
  605. // When opening the DB, all the seqID should be preserved.
  606. Open(names, {});
  607. ASSERT_EQ("v4", Get(1, "foo"));
  608. ASSERT_EQ("v3", Get(1, "bar"));
  609. Close();
  610. db_options_.env = env_;
  611. }
  612. TEST_P(FlushEmptyCFTestWithParam, FlushEmptyCFTest2) {
  613. std::unique_ptr<FaultInjectionTestEnv> fault_env(
  614. new FaultInjectionTestEnv(env_));
  615. db_options_.env = fault_env.get();
  616. db_options_.allow_2pc = allow_2pc_;
  617. Open();
  618. CreateColumnFamilies({"one", "two"});
  619. // Generate log file A.
  620. ASSERT_OK(Put(1, "foo", "v1")); // seqID 1
  621. Reopen();
  622. // Log file A is not dropped after reopening because default column family's
  623. // min log number is 0.
  624. // It flushes to SST file X
  625. ASSERT_OK(Put(1, "foo", "v1")); // seqID 2
  626. ASSERT_OK(Put(1, "bar", "v2")); // seqID 3
  627. // Current log file is file B now. While flushing, a new log file C is created
  628. // and is set to current. Both CFs' min log number is set to file C so after
  629. // flushing file B is deleted. Log file A still remains.
  630. // Flushed to SST file Y.
  631. Flush(1);
  632. ASSERT_OK(Put(0, "bar", "v2")); // seqID 4
  633. ASSERT_OK(Put(2, "bar", "v2")); // seqID 5
  634. ASSERT_OK(Put(1, "bar", "v3")); // seqID 6
  635. // Flushing all column families. This forces all CFs' min log to current. This
  636. // is written to the manifest file. Log file C is cleared.
  637. Flush(0);
  638. Flush(1);
  639. Flush(2);
  640. // Write to log file D
  641. ASSERT_OK(Put(1, "bar", "v4")); // seqID 7
  642. ASSERT_OK(Put(1, "bar", "v5")); // seqID 8
  643. db_->FlushWAL(false);
  644. // Preserve file system state up to here to simulate a crash condition.
  645. fault_env->SetFilesystemActive(false);
  646. std::vector<std::string> names;
  647. for (auto name : names_) {
  648. if (name != "") {
  649. names.push_back(name);
  650. }
  651. }
  652. Close();
  653. fault_env->ResetState();
  654. // Before opening, there are two logfiles:
  655. // Log file A contains seqID 1
  656. // Log file D contains seqID 7, 8
  657. // Min log number:
  658. // default CF: D
  659. // CF one, two: D
  660. // When opening the DB, log file D should be replayed using the seqID
  661. // specified in the file.
  662. Open(names, {});
  663. ASSERT_EQ("v1", Get(1, "foo"));
  664. ASSERT_EQ("v5", Get(1, "bar"));
  665. Close();
  666. db_options_.env = env_;
  667. }
  668. INSTANTIATE_TEST_CASE_P(
  669. FormatDef, FlushEmptyCFTestWithParam,
  670. testing::Values(std::make_tuple(test::kDefaultFormatVersion, true),
  671. std::make_tuple(test::kDefaultFormatVersion, false)));
  672. INSTANTIATE_TEST_CASE_P(
  673. FormatLatest, FlushEmptyCFTestWithParam,
  674. testing::Values(std::make_tuple(test::kLatestFormatVersion, true),
  675. std::make_tuple(test::kLatestFormatVersion, false)));
  676. TEST_P(ColumnFamilyTest, AddDrop) {
  677. Open();
  678. CreateColumnFamilies({"one", "two", "three"});
  679. ASSERT_EQ("NOT_FOUND", Get(1, "fodor"));
  680. ASSERT_EQ("NOT_FOUND", Get(2, "fodor"));
  681. DropColumnFamilies({2});
  682. ASSERT_EQ("NOT_FOUND", Get(1, "fodor"));
  683. CreateColumnFamilies({"four"});
  684. ASSERT_EQ("NOT_FOUND", Get(3, "fodor"));
  685. ASSERT_OK(Put(1, "fodor", "mirko"));
  686. ASSERT_EQ("mirko", Get(1, "fodor"));
  687. ASSERT_EQ("NOT_FOUND", Get(3, "fodor"));
  688. Close();
  689. ASSERT_TRUE(TryOpen({"default"}).IsInvalidArgument());
  690. Open({"default", "one", "three", "four"});
  691. DropColumnFamilies({1});
  692. Reopen();
  693. Close();
  694. std::vector<std::string> families;
  695. ASSERT_OK(DB::ListColumnFamilies(db_options_, dbname_, &families));
  696. std::sort(families.begin(), families.end());
  697. ASSERT_TRUE(families ==
  698. std::vector<std::string>({"default", "four", "three"}));
  699. }
  700. TEST_P(ColumnFamilyTest, BulkAddDrop) {
  701. constexpr int kNumCF = 1000;
  702. ColumnFamilyOptions cf_options;
  703. WriteOptions write_options;
  704. Open();
  705. std::vector<std::string> cf_names;
  706. std::vector<ColumnFamilyHandle*> cf_handles;
  707. for (int i = 1; i <= kNumCF; i++) {
  708. cf_names.push_back("cf1-" + ToString(i));
  709. }
  710. ASSERT_OK(db_->CreateColumnFamilies(cf_options, cf_names, &cf_handles));
  711. for (int i = 1; i <= kNumCF; i++) {
  712. ASSERT_OK(db_->Put(write_options, cf_handles[i - 1], "foo", "bar"));
  713. }
  714. ASSERT_OK(db_->DropColumnFamilies(cf_handles));
  715. std::vector<ColumnFamilyDescriptor> cf_descriptors;
  716. for (auto* handle : cf_handles) {
  717. delete handle;
  718. }
  719. cf_handles.clear();
  720. for (int i = 1; i <= kNumCF; i++) {
  721. cf_descriptors.emplace_back("cf2-" + ToString(i), ColumnFamilyOptions());
  722. }
  723. ASSERT_OK(db_->CreateColumnFamilies(cf_descriptors, &cf_handles));
  724. for (int i = 1; i <= kNumCF; i++) {
  725. ASSERT_OK(db_->Put(write_options, cf_handles[i - 1], "foo", "bar"));
  726. }
  727. ASSERT_OK(db_->DropColumnFamilies(cf_handles));
  728. for (auto* handle : cf_handles) {
  729. delete handle;
  730. }
  731. Close();
  732. std::vector<std::string> families;
  733. ASSERT_OK(DB::ListColumnFamilies(db_options_, dbname_, &families));
  734. std::sort(families.begin(), families.end());
  735. ASSERT_TRUE(families == std::vector<std::string>({"default"}));
  736. }
  737. TEST_P(ColumnFamilyTest, DropTest) {
  738. // first iteration - dont reopen DB before dropping
  739. // second iteration - reopen DB before dropping
  740. for (int iter = 0; iter < 2; ++iter) {
  741. Open({"default"});
  742. CreateColumnFamiliesAndReopen({"pikachu"});
  743. for (int i = 0; i < 100; ++i) {
  744. ASSERT_OK(Put(1, ToString(i), "bar" + ToString(i)));
  745. }
  746. ASSERT_OK(Flush(1));
  747. if (iter == 1) {
  748. Reopen();
  749. }
  750. ASSERT_EQ("bar1", Get(1, "1"));
  751. AssertCountLiveFiles(1);
  752. DropColumnFamilies({1});
  753. // make sure that all files are deleted when we drop the column family
  754. AssertCountLiveFiles(0);
  755. Destroy();
  756. }
  757. }
  758. TEST_P(ColumnFamilyTest, WriteBatchFailure) {
  759. Open();
  760. CreateColumnFamiliesAndReopen({"one", "two"});
  761. WriteBatch batch;
  762. batch.Put(handles_[0], Slice("existing"), Slice("column-family"));
  763. batch.Put(handles_[1], Slice("non-existing"), Slice("column-family"));
  764. ASSERT_OK(db_->Write(WriteOptions(), &batch));
  765. DropColumnFamilies({1});
  766. WriteOptions woptions_ignore_missing_cf;
  767. woptions_ignore_missing_cf.ignore_missing_column_families = true;
  768. batch.Put(handles_[0], Slice("still here"), Slice("column-family"));
  769. ASSERT_OK(db_->Write(woptions_ignore_missing_cf, &batch));
  770. ASSERT_EQ("column-family", Get(0, "still here"));
  771. Status s = db_->Write(WriteOptions(), &batch);
  772. ASSERT_TRUE(s.IsInvalidArgument());
  773. Close();
  774. }
  775. TEST_P(ColumnFamilyTest, ReadWrite) {
  776. Open();
  777. CreateColumnFamiliesAndReopen({"one", "two"});
  778. ASSERT_OK(Put(0, "foo", "v1"));
  779. ASSERT_OK(Put(0, "bar", "v2"));
  780. ASSERT_OK(Put(1, "mirko", "v3"));
  781. ASSERT_OK(Put(0, "foo", "v2"));
  782. ASSERT_OK(Put(2, "fodor", "v5"));
  783. for (int iter = 0; iter <= 3; ++iter) {
  784. ASSERT_EQ("v2", Get(0, "foo"));
  785. ASSERT_EQ("v2", Get(0, "bar"));
  786. ASSERT_EQ("v3", Get(1, "mirko"));
  787. ASSERT_EQ("v5", Get(2, "fodor"));
  788. ASSERT_EQ("NOT_FOUND", Get(0, "fodor"));
  789. ASSERT_EQ("NOT_FOUND", Get(1, "fodor"));
  790. ASSERT_EQ("NOT_FOUND", Get(2, "foo"));
  791. if (iter <= 1) {
  792. Reopen();
  793. }
  794. }
  795. Close();
  796. }
  797. TEST_P(ColumnFamilyTest, IgnoreRecoveredLog) {
  798. std::string backup_logs = dbname_ + "/backup_logs";
  799. // delete old files in backup_logs directory
  800. ASSERT_OK(env_->CreateDirIfMissing(dbname_));
  801. ASSERT_OK(env_->CreateDirIfMissing(backup_logs));
  802. std::vector<std::string> old_files;
  803. env_->GetChildren(backup_logs, &old_files);
  804. for (auto& file : old_files) {
  805. if (file != "." && file != "..") {
  806. env_->DeleteFile(backup_logs + "/" + file);
  807. }
  808. }
  809. column_family_options_.merge_operator =
  810. MergeOperators::CreateUInt64AddOperator();
  811. db_options_.wal_dir = dbname_ + "/logs";
  812. Destroy();
  813. Open();
  814. CreateColumnFamilies({"cf1", "cf2"});
  815. // fill up the DB
  816. std::string one, two, three;
  817. PutFixed64(&one, 1);
  818. PutFixed64(&two, 2);
  819. PutFixed64(&three, 3);
  820. ASSERT_OK(Merge(0, "foo", one));
  821. ASSERT_OK(Merge(1, "mirko", one));
  822. ASSERT_OK(Merge(0, "foo", one));
  823. ASSERT_OK(Merge(2, "bla", one));
  824. ASSERT_OK(Merge(2, "fodor", one));
  825. ASSERT_OK(Merge(0, "bar", one));
  826. ASSERT_OK(Merge(2, "bla", one));
  827. ASSERT_OK(Merge(1, "mirko", two));
  828. ASSERT_OK(Merge(1, "franjo", one));
  829. // copy the logs to backup
  830. std::vector<std::string> logs;
  831. env_->GetChildren(db_options_.wal_dir, &logs);
  832. for (auto& log : logs) {
  833. if (log != ".." && log != ".") {
  834. CopyFile(db_options_.wal_dir + "/" + log, backup_logs + "/" + log);
  835. }
  836. }
  837. // recover the DB
  838. Close();
  839. // 1. check consistency
  840. // 2. copy the logs from backup back to WAL dir. if the recovery happens
  841. // again on the same log files, this should lead to incorrect results
  842. // due to applying merge operator twice
  843. // 3. check consistency
  844. for (int iter = 0; iter < 2; ++iter) {
  845. // assert consistency
  846. Open({"default", "cf1", "cf2"});
  847. ASSERT_EQ(two, Get(0, "foo"));
  848. ASSERT_EQ(one, Get(0, "bar"));
  849. ASSERT_EQ(three, Get(1, "mirko"));
  850. ASSERT_EQ(one, Get(1, "franjo"));
  851. ASSERT_EQ(one, Get(2, "fodor"));
  852. ASSERT_EQ(two, Get(2, "bla"));
  853. Close();
  854. if (iter == 0) {
  855. // copy the logs from backup back to wal dir
  856. for (auto& log : logs) {
  857. if (log != ".." && log != ".") {
  858. CopyFile(backup_logs + "/" + log, db_options_.wal_dir + "/" + log);
  859. }
  860. }
  861. }
  862. }
  863. }
  864. #ifndef ROCKSDB_LITE // TEST functions used are not supported
  865. TEST_P(ColumnFamilyTest, FlushTest) {
  866. Open();
  867. CreateColumnFamiliesAndReopen({"one", "two"});
  868. ASSERT_OK(Put(0, "foo", "v1"));
  869. ASSERT_OK(Put(0, "bar", "v2"));
  870. ASSERT_OK(Put(1, "mirko", "v3"));
  871. ASSERT_OK(Put(0, "foo", "v2"));
  872. ASSERT_OK(Put(2, "fodor", "v5"));
  873. for (int j = 0; j < 2; j++) {
  874. ReadOptions ro;
  875. std::vector<Iterator*> iterators;
  876. // Hold super version.
  877. if (j == 0) {
  878. ASSERT_OK(db_->NewIterators(ro, handles_, &iterators));
  879. }
  880. for (int i = 0; i < 3; ++i) {
  881. uint64_t max_total_in_memory_state =
  882. MaxTotalInMemoryState();
  883. Flush(i);
  884. AssertMaxTotalInMemoryState(max_total_in_memory_state);
  885. }
  886. ASSERT_OK(Put(1, "foofoo", "bar"));
  887. ASSERT_OK(Put(0, "foofoo", "bar"));
  888. for (auto* it : iterators) {
  889. delete it;
  890. }
  891. }
  892. Reopen();
  893. for (int iter = 0; iter <= 2; ++iter) {
  894. ASSERT_EQ("v2", Get(0, "foo"));
  895. ASSERT_EQ("v2", Get(0, "bar"));
  896. ASSERT_EQ("v3", Get(1, "mirko"));
  897. ASSERT_EQ("v5", Get(2, "fodor"));
  898. ASSERT_EQ("NOT_FOUND", Get(0, "fodor"));
  899. ASSERT_EQ("NOT_FOUND", Get(1, "fodor"));
  900. ASSERT_EQ("NOT_FOUND", Get(2, "foo"));
  901. if (iter <= 1) {
  902. Reopen();
  903. }
  904. }
  905. Close();
  906. }
  907. // Makes sure that obsolete log files get deleted
  908. TEST_P(ColumnFamilyTest, LogDeletionTest) {
  909. db_options_.max_total_wal_size = std::numeric_limits<uint64_t>::max();
  910. column_family_options_.arena_block_size = 4 * 1024;
  911. column_family_options_.write_buffer_size = 128000; // 128KB
  912. Open();
  913. CreateColumnFamilies({"one", "two", "three", "four"});
  914. // Each bracket is one log file. if number is in (), it means
  915. // we don't need it anymore (it's been flushed)
  916. // []
  917. AssertCountLiveLogFiles(0);
  918. PutRandomData(0, 1, 128);
  919. // [0]
  920. PutRandomData(1, 1, 128);
  921. // [0, 1]
  922. PutRandomData(1, 1000, 128);
  923. WaitForFlush(1);
  924. // [0, (1)] [1]
  925. AssertCountLiveLogFiles(2);
  926. PutRandomData(0, 1, 128);
  927. // [0, (1)] [0, 1]
  928. AssertCountLiveLogFiles(2);
  929. PutRandomData(2, 1, 128);
  930. // [0, (1)] [0, 1, 2]
  931. PutRandomData(2, 1000, 128);
  932. WaitForFlush(2);
  933. // [0, (1)] [0, 1, (2)] [2]
  934. AssertCountLiveLogFiles(3);
  935. PutRandomData(2, 1000, 128);
  936. WaitForFlush(2);
  937. // [0, (1)] [0, 1, (2)] [(2)] [2]
  938. AssertCountLiveLogFiles(4);
  939. PutRandomData(3, 1, 128);
  940. // [0, (1)] [0, 1, (2)] [(2)] [2, 3]
  941. PutRandomData(1, 1, 128);
  942. // [0, (1)] [0, 1, (2)] [(2)] [1, 2, 3]
  943. AssertCountLiveLogFiles(4);
  944. PutRandomData(1, 1000, 128);
  945. WaitForFlush(1);
  946. // [0, (1)] [0, (1), (2)] [(2)] [(1), 2, 3] [1]
  947. AssertCountLiveLogFiles(5);
  948. PutRandomData(0, 1000, 128);
  949. WaitForFlush(0);
  950. // [(0), (1)] [(0), (1), (2)] [(2)] [(1), 2, 3] [1, (0)] [0]
  951. // delete obsolete logs -->
  952. // [(1), 2, 3] [1, (0)] [0]
  953. AssertCountLiveLogFiles(3);
  954. PutRandomData(0, 1000, 128);
  955. WaitForFlush(0);
  956. // [(1), 2, 3] [1, (0)], [(0)] [0]
  957. AssertCountLiveLogFiles(4);
  958. PutRandomData(1, 1000, 128);
  959. WaitForFlush(1);
  960. // [(1), 2, 3] [(1), (0)] [(0)] [0, (1)] [1]
  961. AssertCountLiveLogFiles(5);
  962. PutRandomData(2, 1000, 128);
  963. WaitForFlush(2);
  964. // [(1), (2), 3] [(1), (0)] [(0)] [0, (1)] [1, (2)], [2]
  965. AssertCountLiveLogFiles(6);
  966. PutRandomData(3, 1000, 128);
  967. WaitForFlush(3);
  968. // [(1), (2), (3)] [(1), (0)] [(0)] [0, (1)] [1, (2)], [2, (3)] [3]
  969. // delete obsolete logs -->
  970. // [0, (1)] [1, (2)], [2, (3)] [3]
  971. AssertCountLiveLogFiles(4);
  972. Close();
  973. }
  974. #endif // !ROCKSDB_LITE
  975. TEST_P(ColumnFamilyTest, CrashAfterFlush) {
  976. std::unique_ptr<FaultInjectionTestEnv> fault_env(
  977. new FaultInjectionTestEnv(env_));
  978. db_options_.env = fault_env.get();
  979. Open();
  980. CreateColumnFamilies({"one"});
  981. WriteBatch batch;
  982. batch.Put(handles_[0], Slice("foo"), Slice("bar"));
  983. batch.Put(handles_[1], Slice("foo"), Slice("bar"));
  984. ASSERT_OK(db_->Write(WriteOptions(), &batch));
  985. Flush(0);
  986. fault_env->SetFilesystemActive(false);
  987. std::vector<std::string> names;
  988. for (auto name : names_) {
  989. if (name != "") {
  990. names.push_back(name);
  991. }
  992. }
  993. Close();
  994. fault_env->DropUnsyncedFileData();
  995. fault_env->ResetState();
  996. Open(names, {});
  997. // Write batch should be atomic.
  998. ASSERT_EQ(Get(0, "foo"), Get(1, "foo"));
  999. Close();
  1000. db_options_.env = env_;
  1001. }
  1002. TEST_P(ColumnFamilyTest, OpenNonexistentColumnFamily) {
  1003. ASSERT_OK(TryOpen({"default"}));
  1004. Close();
  1005. ASSERT_TRUE(TryOpen({"default", "dne"}).IsInvalidArgument());
  1006. }
  1007. #ifndef ROCKSDB_LITE // WaitForFlush() is not supported
  1008. // Makes sure that obsolete log files get deleted
  1009. TEST_P(ColumnFamilyTest, DifferentWriteBufferSizes) {
  1010. // disable flushing stale column families
  1011. db_options_.max_total_wal_size = std::numeric_limits<uint64_t>::max();
  1012. Open();
  1013. CreateColumnFamilies({"one", "two", "three"});
  1014. ColumnFamilyOptions default_cf, one, two, three;
  1015. // setup options. all column families have max_write_buffer_number setup to 10
  1016. // "default" -> 100KB memtable, start flushing immediatelly
  1017. // "one" -> 200KB memtable, start flushing with two immutable memtables
  1018. // "two" -> 1MB memtable, start flushing with three immutable memtables
  1019. // "three" -> 90KB memtable, start flushing with four immutable memtables
  1020. default_cf.write_buffer_size = 100000;
  1021. default_cf.arena_block_size = 4 * 4096;
  1022. default_cf.max_write_buffer_number = 10;
  1023. default_cf.min_write_buffer_number_to_merge = 1;
  1024. default_cf.max_write_buffer_size_to_maintain = 0;
  1025. one.write_buffer_size = 200000;
  1026. one.arena_block_size = 4 * 4096;
  1027. one.max_write_buffer_number = 10;
  1028. one.min_write_buffer_number_to_merge = 2;
  1029. one.max_write_buffer_size_to_maintain =
  1030. static_cast<int>(one.write_buffer_size);
  1031. two.write_buffer_size = 1000000;
  1032. two.arena_block_size = 4 * 4096;
  1033. two.max_write_buffer_number = 10;
  1034. two.min_write_buffer_number_to_merge = 3;
  1035. two.max_write_buffer_size_to_maintain =
  1036. static_cast<int>(two.write_buffer_size);
  1037. three.write_buffer_size = 4096 * 22;
  1038. three.arena_block_size = 4096;
  1039. three.max_write_buffer_number = 10;
  1040. three.min_write_buffer_number_to_merge = 4;
  1041. three.max_write_buffer_size_to_maintain =
  1042. static_cast<int>(three.write_buffer_size);
  1043. Reopen({default_cf, one, two, three});
  1044. int micros_wait_for_flush = 10000;
  1045. PutRandomData(0, 100, 1000);
  1046. WaitForFlush(0);
  1047. AssertNumberOfImmutableMemtables({0, 0, 0, 0});
  1048. AssertCountLiveLogFiles(1);
  1049. PutRandomData(1, 200, 1000);
  1050. env_->SleepForMicroseconds(micros_wait_for_flush);
  1051. AssertNumberOfImmutableMemtables({0, 1, 0, 0});
  1052. AssertCountLiveLogFiles(2);
  1053. PutRandomData(2, 1000, 1000);
  1054. env_->SleepForMicroseconds(micros_wait_for_flush);
  1055. AssertNumberOfImmutableMemtables({0, 1, 1, 0});
  1056. AssertCountLiveLogFiles(3);
  1057. PutRandomData(2, 1000, 1000);
  1058. env_->SleepForMicroseconds(micros_wait_for_flush);
  1059. AssertNumberOfImmutableMemtables({0, 1, 2, 0});
  1060. AssertCountLiveLogFiles(4);
  1061. PutRandomData(3, 93, 990);
  1062. env_->SleepForMicroseconds(micros_wait_for_flush);
  1063. AssertNumberOfImmutableMemtables({0, 1, 2, 1});
  1064. AssertCountLiveLogFiles(5);
  1065. PutRandomData(3, 88, 990);
  1066. env_->SleepForMicroseconds(micros_wait_for_flush);
  1067. AssertNumberOfImmutableMemtables({0, 1, 2, 2});
  1068. AssertCountLiveLogFiles(6);
  1069. PutRandomData(3, 88, 990);
  1070. env_->SleepForMicroseconds(micros_wait_for_flush);
  1071. AssertNumberOfImmutableMemtables({0, 1, 2, 3});
  1072. AssertCountLiveLogFiles(7);
  1073. PutRandomData(0, 100, 1000);
  1074. WaitForFlush(0);
  1075. AssertNumberOfImmutableMemtables({0, 1, 2, 3});
  1076. AssertCountLiveLogFiles(8);
  1077. PutRandomData(2, 100, 10000);
  1078. WaitForFlush(2);
  1079. AssertNumberOfImmutableMemtables({0, 1, 0, 3});
  1080. AssertCountLiveLogFiles(9);
  1081. PutRandomData(3, 88, 990);
  1082. WaitForFlush(3);
  1083. AssertNumberOfImmutableMemtables({0, 1, 0, 0});
  1084. AssertCountLiveLogFiles(10);
  1085. PutRandomData(3, 88, 990);
  1086. env_->SleepForMicroseconds(micros_wait_for_flush);
  1087. AssertNumberOfImmutableMemtables({0, 1, 0, 1});
  1088. AssertCountLiveLogFiles(11);
  1089. PutRandomData(1, 200, 1000);
  1090. WaitForFlush(1);
  1091. AssertNumberOfImmutableMemtables({0, 0, 0, 1});
  1092. AssertCountLiveLogFiles(5);
  1093. PutRandomData(3, 88 * 3, 990);
  1094. WaitForFlush(3);
  1095. PutRandomData(3, 88 * 4, 990);
  1096. WaitForFlush(3);
  1097. AssertNumberOfImmutableMemtables({0, 0, 0, 0});
  1098. AssertCountLiveLogFiles(12);
  1099. PutRandomData(0, 100, 1000);
  1100. WaitForFlush(0);
  1101. AssertNumberOfImmutableMemtables({0, 0, 0, 0});
  1102. AssertCountLiveLogFiles(12);
  1103. PutRandomData(2, 3 * 1000, 1000);
  1104. WaitForFlush(2);
  1105. AssertNumberOfImmutableMemtables({0, 0, 0, 0});
  1106. AssertCountLiveLogFiles(12);
  1107. PutRandomData(1, 2*200, 1000);
  1108. WaitForFlush(1);
  1109. AssertNumberOfImmutableMemtables({0, 0, 0, 0});
  1110. AssertCountLiveLogFiles(7);
  1111. Close();
  1112. }
  1113. #endif // !ROCKSDB_LITE
  1114. // The test is commented out because we want to test that snapshot is
  1115. // not created for memtables not supported it, but There isn't a memtable
  1116. // that doesn't support snapshot right now. If we have one later, we can
  1117. // re-enable the test.
  1118. //
  1119. // #ifndef ROCKSDB_LITE // Cuckoo is not supported in lite
  1120. // TEST_P(ColumnFamilyTest, MemtableNotSupportSnapshot) {
  1121. // db_options_.allow_concurrent_memtable_write = false;
  1122. // Open();
  1123. // auto* s1 = dbfull()->GetSnapshot();
  1124. // ASSERT_TRUE(s1 != nullptr);
  1125. // dbfull()->ReleaseSnapshot(s1);
  1126. // // Add a column family that doesn't support snapshot
  1127. // ColumnFamilyOptions first;
  1128. // first.memtable_factory.reset(new DummyMemtableNotSupportingSnapshot());
  1129. // CreateColumnFamilies({"first"}, {first});
  1130. // auto* s2 = dbfull()->GetSnapshot();
  1131. // ASSERT_TRUE(s2 == nullptr);
  1132. // // Add a column family that supports snapshot. Snapshot stays not
  1133. // supported. ColumnFamilyOptions second; CreateColumnFamilies({"second"},
  1134. // {second}); auto* s3 = dbfull()->GetSnapshot(); ASSERT_TRUE(s3 == nullptr);
  1135. // Close();
  1136. // }
  1137. // #endif // !ROCKSDB_LITE
  1138. class TestComparator : public Comparator {
  1139. int Compare(const ROCKSDB_NAMESPACE::Slice& /*a*/,
  1140. const ROCKSDB_NAMESPACE::Slice& /*b*/) const override {
  1141. return 0;
  1142. }
  1143. const char* Name() const override { return "Test"; }
  1144. void FindShortestSeparator(
  1145. std::string* /*start*/,
  1146. const ROCKSDB_NAMESPACE::Slice& /*limit*/) const override {}
  1147. void FindShortSuccessor(std::string* /*key*/) const override {}
  1148. };
  1149. static TestComparator third_comparator;
  1150. static TestComparator fourth_comparator;
  1151. // Test that we can retrieve the comparator from a created CF
  1152. TEST_P(ColumnFamilyTest, GetComparator) {
  1153. Open();
  1154. // Add a column family with no comparator specified
  1155. CreateColumnFamilies({"first"});
  1156. const Comparator* comp = handles_[0]->GetComparator();
  1157. ASSERT_EQ(comp, BytewiseComparator());
  1158. // Add three column families - one with no comparator and two
  1159. // with comparators specified
  1160. ColumnFamilyOptions second, third, fourth;
  1161. second.comparator = &third_comparator;
  1162. third.comparator = &fourth_comparator;
  1163. CreateColumnFamilies({"second", "third", "fourth"}, {second, third, fourth});
  1164. ASSERT_EQ(handles_[1]->GetComparator(), BytewiseComparator());
  1165. ASSERT_EQ(handles_[2]->GetComparator(), &third_comparator);
  1166. ASSERT_EQ(handles_[3]->GetComparator(), &fourth_comparator);
  1167. Close();
  1168. }
  1169. TEST_P(ColumnFamilyTest, DifferentMergeOperators) {
  1170. Open();
  1171. CreateColumnFamilies({"first", "second"});
  1172. ColumnFamilyOptions default_cf, first, second;
  1173. first.merge_operator = MergeOperators::CreateUInt64AddOperator();
  1174. second.merge_operator = MergeOperators::CreateStringAppendOperator();
  1175. Reopen({default_cf, first, second});
  1176. std::string one, two, three;
  1177. PutFixed64(&one, 1);
  1178. PutFixed64(&two, 2);
  1179. PutFixed64(&three, 3);
  1180. ASSERT_OK(Put(0, "foo", two));
  1181. ASSERT_OK(Put(0, "foo", one));
  1182. ASSERT_TRUE(Merge(0, "foo", two).IsNotSupported());
  1183. ASSERT_EQ(Get(0, "foo"), one);
  1184. ASSERT_OK(Put(1, "foo", two));
  1185. ASSERT_OK(Put(1, "foo", one));
  1186. ASSERT_OK(Merge(1, "foo", two));
  1187. ASSERT_EQ(Get(1, "foo"), three);
  1188. ASSERT_OK(Put(2, "foo", two));
  1189. ASSERT_OK(Put(2, "foo", one));
  1190. ASSERT_OK(Merge(2, "foo", two));
  1191. ASSERT_EQ(Get(2, "foo"), one + "," + two);
  1192. Close();
  1193. }
  1194. #ifndef ROCKSDB_LITE // WaitForFlush() is not supported
  1195. TEST_P(ColumnFamilyTest, DifferentCompactionStyles) {
  1196. Open();
  1197. CreateColumnFamilies({"one", "two"});
  1198. ColumnFamilyOptions default_cf, one, two;
  1199. db_options_.max_open_files = 20; // only 10 files in file cache
  1200. default_cf.compaction_style = kCompactionStyleLevel;
  1201. default_cf.num_levels = 3;
  1202. default_cf.write_buffer_size = 64 << 10; // 64KB
  1203. default_cf.target_file_size_base = 30 << 10;
  1204. default_cf.max_compaction_bytes = static_cast<uint64_t>(1) << 60;
  1205. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1206. table_options.no_block_cache = true;
  1207. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1208. one.compaction_style = kCompactionStyleUniversal;
  1209. one.num_levels = 1;
  1210. // trigger compaction if there are >= 4 files
  1211. one.level0_file_num_compaction_trigger = 4;
  1212. one.write_buffer_size = 120000;
  1213. two.compaction_style = kCompactionStyleLevel;
  1214. two.num_levels = 4;
  1215. two.level0_file_num_compaction_trigger = 3;
  1216. two.write_buffer_size = 100000;
  1217. Reopen({default_cf, one, two});
  1218. // SETUP column family "one" -- universal style
  1219. for (int i = 0; i < one.level0_file_num_compaction_trigger - 1; ++i) {
  1220. PutRandomData(1, 10, 12000);
  1221. PutRandomData(1, 1, 10);
  1222. WaitForFlush(1);
  1223. AssertFilesPerLevel(ToString(i + 1), 1);
  1224. }
  1225. // SETUP column family "two" -- level style with 4 levels
  1226. for (int i = 0; i < two.level0_file_num_compaction_trigger - 1; ++i) {
  1227. PutRandomData(2, 10, 12000);
  1228. PutRandomData(2, 1, 10);
  1229. WaitForFlush(2);
  1230. AssertFilesPerLevel(ToString(i + 1), 2);
  1231. }
  1232. // TRIGGER compaction "one"
  1233. PutRandomData(1, 10, 12000);
  1234. PutRandomData(1, 1, 10);
  1235. // TRIGGER compaction "two"
  1236. PutRandomData(2, 10, 12000);
  1237. PutRandomData(2, 1, 10);
  1238. // WAIT for compactions
  1239. WaitForCompaction();
  1240. // VERIFY compaction "one"
  1241. AssertFilesPerLevel("1", 1);
  1242. // VERIFY compaction "two"
  1243. AssertFilesPerLevel("0,1", 2);
  1244. CompactAll(2);
  1245. AssertFilesPerLevel("0,1", 2);
  1246. Close();
  1247. }
  1248. #endif // !ROCKSDB_LITE
  1249. #ifndef ROCKSDB_LITE
  1250. // Sync points not supported in RocksDB Lite
  1251. TEST_P(ColumnFamilyTest, MultipleManualCompactions) {
  1252. Open();
  1253. CreateColumnFamilies({"one", "two"});
  1254. ColumnFamilyOptions default_cf, one, two;
  1255. db_options_.max_open_files = 20; // only 10 files in file cache
  1256. db_options_.max_background_compactions = 3;
  1257. default_cf.compaction_style = kCompactionStyleLevel;
  1258. default_cf.num_levels = 3;
  1259. default_cf.write_buffer_size = 64 << 10; // 64KB
  1260. default_cf.target_file_size_base = 30 << 10;
  1261. default_cf.max_compaction_bytes = default_cf.target_file_size_base * 1100;
  1262. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1263. table_options.no_block_cache = true;
  1264. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1265. one.compaction_style = kCompactionStyleUniversal;
  1266. one.num_levels = 1;
  1267. // trigger compaction if there are >= 4 files
  1268. one.level0_file_num_compaction_trigger = 4;
  1269. one.write_buffer_size = 120000;
  1270. two.compaction_style = kCompactionStyleLevel;
  1271. two.num_levels = 4;
  1272. two.level0_file_num_compaction_trigger = 3;
  1273. two.write_buffer_size = 100000;
  1274. Reopen({default_cf, one, two});
  1275. // SETUP column family "one" -- universal style
  1276. for (int i = 0; i < one.level0_file_num_compaction_trigger - 2; ++i) {
  1277. PutRandomData(1, 10, 12000, true);
  1278. PutRandomData(1, 1, 10, true);
  1279. WaitForFlush(1);
  1280. AssertFilesPerLevel(ToString(i + 1), 1);
  1281. }
  1282. bool cf_1_1 = true;
  1283. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  1284. {{"ColumnFamilyTest::MultiManual:4", "ColumnFamilyTest::MultiManual:1"},
  1285. {"ColumnFamilyTest::MultiManual:2", "ColumnFamilyTest::MultiManual:5"},
  1286. {"ColumnFamilyTest::MultiManual:2", "ColumnFamilyTest::MultiManual:3"}});
  1287. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  1288. "DBImpl::BackgroundCompaction:NonTrivial:AfterRun", [&](void* /*arg*/) {
  1289. if (cf_1_1) {
  1290. TEST_SYNC_POINT("ColumnFamilyTest::MultiManual:4");
  1291. cf_1_1 = false;
  1292. TEST_SYNC_POINT("ColumnFamilyTest::MultiManual:3");
  1293. }
  1294. });
  1295. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  1296. std::vector<port::Thread> threads;
  1297. threads.emplace_back([&] {
  1298. CompactRangeOptions compact_options;
  1299. compact_options.exclusive_manual_compaction = false;
  1300. ASSERT_OK(
  1301. db_->CompactRange(compact_options, handles_[1], nullptr, nullptr));
  1302. });
  1303. // SETUP column family "two" -- level style with 4 levels
  1304. for (int i = 0; i < two.level0_file_num_compaction_trigger - 2; ++i) {
  1305. PutRandomData(2, 10, 12000);
  1306. PutRandomData(2, 1, 10);
  1307. WaitForFlush(2);
  1308. AssertFilesPerLevel(ToString(i + 1), 2);
  1309. }
  1310. threads.emplace_back([&] {
  1311. TEST_SYNC_POINT("ColumnFamilyTest::MultiManual:1");
  1312. CompactRangeOptions compact_options;
  1313. compact_options.exclusive_manual_compaction = false;
  1314. ASSERT_OK(
  1315. db_->CompactRange(compact_options, handles_[2], nullptr, nullptr));
  1316. TEST_SYNC_POINT("ColumnFamilyTest::MultiManual:2");
  1317. });
  1318. TEST_SYNC_POINT("ColumnFamilyTest::MultiManual:5");
  1319. for (auto& t : threads) {
  1320. t.join();
  1321. }
  1322. // VERIFY compaction "one"
  1323. AssertFilesPerLevel("1", 1);
  1324. // VERIFY compaction "two"
  1325. AssertFilesPerLevel("0,1", 2);
  1326. CompactAll(2);
  1327. AssertFilesPerLevel("0,1", 2);
  1328. // Compare against saved keys
  1329. std::set<std::string>::iterator key_iter = keys_[1].begin();
  1330. while (key_iter != keys_[1].end()) {
  1331. ASSERT_NE("NOT_FOUND", Get(1, *key_iter));
  1332. key_iter++;
  1333. }
  1334. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  1335. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  1336. Close();
  1337. }
  1338. TEST_P(ColumnFamilyTest, AutomaticAndManualCompactions) {
  1339. Open();
  1340. CreateColumnFamilies({"one", "two"});
  1341. ColumnFamilyOptions default_cf, one, two;
  1342. db_options_.max_open_files = 20; // only 10 files in file cache
  1343. db_options_.max_background_compactions = 3;
  1344. default_cf.compaction_style = kCompactionStyleLevel;
  1345. default_cf.num_levels = 3;
  1346. default_cf.write_buffer_size = 64 << 10; // 64KB
  1347. default_cf.target_file_size_base = 30 << 10;
  1348. default_cf.max_compaction_bytes = default_cf.target_file_size_base * 1100;
  1349. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1350. ;
  1351. table_options.no_block_cache = true;
  1352. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1353. one.compaction_style = kCompactionStyleUniversal;
  1354. one.num_levels = 1;
  1355. // trigger compaction if there are >= 4 files
  1356. one.level0_file_num_compaction_trigger = 4;
  1357. one.write_buffer_size = 120000;
  1358. two.compaction_style = kCompactionStyleLevel;
  1359. two.num_levels = 4;
  1360. two.level0_file_num_compaction_trigger = 3;
  1361. two.write_buffer_size = 100000;
  1362. Reopen({default_cf, one, two});
  1363. // make sure all background compaction jobs can be scheduled
  1364. auto stop_token =
  1365. dbfull()->TEST_write_controler().GetCompactionPressureToken();
  1366. bool cf_1_1 = true;
  1367. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  1368. {{"ColumnFamilyTest::AutoManual:4", "ColumnFamilyTest::AutoManual:1"},
  1369. {"ColumnFamilyTest::AutoManual:2", "ColumnFamilyTest::AutoManual:5"},
  1370. {"ColumnFamilyTest::AutoManual:2", "ColumnFamilyTest::AutoManual:3"}});
  1371. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  1372. "DBImpl::BackgroundCompaction:NonTrivial:AfterRun", [&](void* /*arg*/) {
  1373. if (cf_1_1) {
  1374. cf_1_1 = false;
  1375. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:4");
  1376. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:3");
  1377. }
  1378. });
  1379. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  1380. // SETUP column family "one" -- universal style
  1381. for (int i = 0; i < one.level0_file_num_compaction_trigger; ++i) {
  1382. PutRandomData(1, 10, 12000, true);
  1383. PutRandomData(1, 1, 10, true);
  1384. WaitForFlush(1);
  1385. AssertFilesPerLevel(ToString(i + 1), 1);
  1386. }
  1387. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:1");
  1388. // SETUP column family "two" -- level style with 4 levels
  1389. for (int i = 0; i < two.level0_file_num_compaction_trigger - 2; ++i) {
  1390. PutRandomData(2, 10, 12000);
  1391. PutRandomData(2, 1, 10);
  1392. WaitForFlush(2);
  1393. AssertFilesPerLevel(ToString(i + 1), 2);
  1394. }
  1395. ROCKSDB_NAMESPACE::port::Thread threads([&] {
  1396. CompactRangeOptions compact_options;
  1397. compact_options.exclusive_manual_compaction = false;
  1398. ASSERT_OK(
  1399. db_->CompactRange(compact_options, handles_[2], nullptr, nullptr));
  1400. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:2");
  1401. });
  1402. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:5");
  1403. threads.join();
  1404. // WAIT for compactions
  1405. WaitForCompaction();
  1406. // VERIFY compaction "one"
  1407. AssertFilesPerLevel("1", 1);
  1408. // VERIFY compaction "two"
  1409. AssertFilesPerLevel("0,1", 2);
  1410. CompactAll(2);
  1411. AssertFilesPerLevel("0,1", 2);
  1412. // Compare against saved keys
  1413. std::set<std::string>::iterator key_iter = keys_[1].begin();
  1414. while (key_iter != keys_[1].end()) {
  1415. ASSERT_NE("NOT_FOUND", Get(1, *key_iter));
  1416. key_iter++;
  1417. }
  1418. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  1419. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  1420. }
  1421. TEST_P(ColumnFamilyTest, ManualAndAutomaticCompactions) {
  1422. Open();
  1423. CreateColumnFamilies({"one", "two"});
  1424. ColumnFamilyOptions default_cf, one, two;
  1425. db_options_.max_open_files = 20; // only 10 files in file cache
  1426. db_options_.max_background_compactions = 3;
  1427. default_cf.compaction_style = kCompactionStyleLevel;
  1428. default_cf.num_levels = 3;
  1429. default_cf.write_buffer_size = 64 << 10; // 64KB
  1430. default_cf.target_file_size_base = 30 << 10;
  1431. default_cf.max_compaction_bytes = default_cf.target_file_size_base * 1100;
  1432. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1433. ;
  1434. table_options.no_block_cache = true;
  1435. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1436. one.compaction_style = kCompactionStyleUniversal;
  1437. one.num_levels = 1;
  1438. // trigger compaction if there are >= 4 files
  1439. one.level0_file_num_compaction_trigger = 4;
  1440. one.write_buffer_size = 120000;
  1441. two.compaction_style = kCompactionStyleLevel;
  1442. two.num_levels = 4;
  1443. two.level0_file_num_compaction_trigger = 3;
  1444. two.write_buffer_size = 100000;
  1445. Reopen({default_cf, one, two});
  1446. // make sure all background compaction jobs can be scheduled
  1447. auto stop_token =
  1448. dbfull()->TEST_write_controler().GetCompactionPressureToken();
  1449. // SETUP column family "one" -- universal style
  1450. for (int i = 0; i < one.level0_file_num_compaction_trigger - 2; ++i) {
  1451. PutRandomData(1, 10, 12000, true);
  1452. PutRandomData(1, 1, 10, true);
  1453. WaitForFlush(1);
  1454. AssertFilesPerLevel(ToString(i + 1), 1);
  1455. }
  1456. bool cf_1_1 = true;
  1457. bool cf_1_2 = true;
  1458. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  1459. {{"ColumnFamilyTest::ManualAuto:4", "ColumnFamilyTest::ManualAuto:1"},
  1460. {"ColumnFamilyTest::ManualAuto:5", "ColumnFamilyTest::ManualAuto:2"},
  1461. {"ColumnFamilyTest::ManualAuto:2", "ColumnFamilyTest::ManualAuto:3"}});
  1462. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  1463. "DBImpl::BackgroundCompaction:NonTrivial:AfterRun", [&](void* /*arg*/) {
  1464. if (cf_1_1) {
  1465. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:4");
  1466. cf_1_1 = false;
  1467. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:3");
  1468. } else if (cf_1_2) {
  1469. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:2");
  1470. cf_1_2 = false;
  1471. }
  1472. });
  1473. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  1474. ROCKSDB_NAMESPACE::port::Thread threads([&] {
  1475. CompactRangeOptions compact_options;
  1476. compact_options.exclusive_manual_compaction = false;
  1477. ASSERT_OK(
  1478. db_->CompactRange(compact_options, handles_[1], nullptr, nullptr));
  1479. });
  1480. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:1");
  1481. // SETUP column family "two" -- level style with 4 levels
  1482. for (int i = 0; i < two.level0_file_num_compaction_trigger; ++i) {
  1483. PutRandomData(2, 10, 12000);
  1484. PutRandomData(2, 1, 10);
  1485. WaitForFlush(2);
  1486. AssertFilesPerLevel(ToString(i + 1), 2);
  1487. }
  1488. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:5");
  1489. threads.join();
  1490. // WAIT for compactions
  1491. WaitForCompaction();
  1492. // VERIFY compaction "one"
  1493. AssertFilesPerLevel("1", 1);
  1494. // VERIFY compaction "two"
  1495. AssertFilesPerLevel("0,1", 2);
  1496. CompactAll(2);
  1497. AssertFilesPerLevel("0,1", 2);
  1498. // Compare against saved keys
  1499. std::set<std::string>::iterator key_iter = keys_[1].begin();
  1500. while (key_iter != keys_[1].end()) {
  1501. ASSERT_NE("NOT_FOUND", Get(1, *key_iter));
  1502. key_iter++;
  1503. }
  1504. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  1505. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  1506. }
  1507. TEST_P(ColumnFamilyTest, SameCFManualManualCompactions) {
  1508. Open();
  1509. CreateColumnFamilies({"one"});
  1510. ColumnFamilyOptions default_cf, one;
  1511. db_options_.max_open_files = 20; // only 10 files in file cache
  1512. db_options_.max_background_compactions = 3;
  1513. default_cf.compaction_style = kCompactionStyleLevel;
  1514. default_cf.num_levels = 3;
  1515. default_cf.write_buffer_size = 64 << 10; // 64KB
  1516. default_cf.target_file_size_base = 30 << 10;
  1517. default_cf.max_compaction_bytes = default_cf.target_file_size_base * 1100;
  1518. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1519. ;
  1520. table_options.no_block_cache = true;
  1521. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1522. one.compaction_style = kCompactionStyleUniversal;
  1523. one.num_levels = 1;
  1524. // trigger compaction if there are >= 4 files
  1525. one.level0_file_num_compaction_trigger = 4;
  1526. one.write_buffer_size = 120000;
  1527. Reopen({default_cf, one});
  1528. // make sure all background compaction jobs can be scheduled
  1529. auto stop_token =
  1530. dbfull()->TEST_write_controler().GetCompactionPressureToken();
  1531. // SETUP column family "one" -- universal style
  1532. for (int i = 0; i < one.level0_file_num_compaction_trigger - 2; ++i) {
  1533. PutRandomData(1, 10, 12000, true);
  1534. PutRandomData(1, 1, 10, true);
  1535. WaitForFlush(1);
  1536. AssertFilesPerLevel(ToString(i + 1), 1);
  1537. }
  1538. bool cf_1_1 = true;
  1539. bool cf_1_2 = true;
  1540. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  1541. {{"ColumnFamilyTest::ManualManual:4", "ColumnFamilyTest::ManualManual:2"},
  1542. {"ColumnFamilyTest::ManualManual:4", "ColumnFamilyTest::ManualManual:5"},
  1543. {"ColumnFamilyTest::ManualManual:1", "ColumnFamilyTest::ManualManual:2"},
  1544. {"ColumnFamilyTest::ManualManual:1",
  1545. "ColumnFamilyTest::ManualManual:3"}});
  1546. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  1547. "DBImpl::BackgroundCompaction:NonTrivial:AfterRun", [&](void* /*arg*/) {
  1548. if (cf_1_1) {
  1549. TEST_SYNC_POINT("ColumnFamilyTest::ManualManual:4");
  1550. cf_1_1 = false;
  1551. TEST_SYNC_POINT("ColumnFamilyTest::ManualManual:3");
  1552. } else if (cf_1_2) {
  1553. TEST_SYNC_POINT("ColumnFamilyTest::ManualManual:2");
  1554. cf_1_2 = false;
  1555. }
  1556. });
  1557. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  1558. ROCKSDB_NAMESPACE::port::Thread threads([&] {
  1559. CompactRangeOptions compact_options;
  1560. compact_options.exclusive_manual_compaction = true;
  1561. ASSERT_OK(
  1562. db_->CompactRange(compact_options, handles_[1], nullptr, nullptr));
  1563. });
  1564. TEST_SYNC_POINT("ColumnFamilyTest::ManualManual:5");
  1565. WaitForFlush(1);
  1566. // Add more L0 files and force another manual compaction
  1567. for (int i = 0; i < one.level0_file_num_compaction_trigger - 2; ++i) {
  1568. PutRandomData(1, 10, 12000, true);
  1569. PutRandomData(1, 1, 10, true);
  1570. WaitForFlush(1);
  1571. AssertFilesPerLevel(ToString(one.level0_file_num_compaction_trigger + i),
  1572. 1);
  1573. }
  1574. ROCKSDB_NAMESPACE::port::Thread threads1([&] {
  1575. CompactRangeOptions compact_options;
  1576. compact_options.exclusive_manual_compaction = false;
  1577. ASSERT_OK(
  1578. db_->CompactRange(compact_options, handles_[1], nullptr, nullptr));
  1579. });
  1580. TEST_SYNC_POINT("ColumnFamilyTest::ManualManual:1");
  1581. threads.join();
  1582. threads1.join();
  1583. WaitForCompaction();
  1584. // VERIFY compaction "one"
  1585. ASSERT_LE(NumTableFilesAtLevel(0, 1), 2);
  1586. // Compare against saved keys
  1587. std::set<std::string>::iterator key_iter = keys_[1].begin();
  1588. while (key_iter != keys_[1].end()) {
  1589. ASSERT_NE("NOT_FOUND", Get(1, *key_iter));
  1590. key_iter++;
  1591. }
  1592. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  1593. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  1594. }
  1595. TEST_P(ColumnFamilyTest, SameCFManualAutomaticCompactions) {
  1596. Open();
  1597. CreateColumnFamilies({"one"});
  1598. ColumnFamilyOptions default_cf, one;
  1599. db_options_.max_open_files = 20; // only 10 files in file cache
  1600. db_options_.max_background_compactions = 3;
  1601. default_cf.compaction_style = kCompactionStyleLevel;
  1602. default_cf.num_levels = 3;
  1603. default_cf.write_buffer_size = 64 << 10; // 64KB
  1604. default_cf.target_file_size_base = 30 << 10;
  1605. default_cf.max_compaction_bytes = default_cf.target_file_size_base * 1100;
  1606. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1607. ;
  1608. table_options.no_block_cache = true;
  1609. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1610. one.compaction_style = kCompactionStyleUniversal;
  1611. one.num_levels = 1;
  1612. // trigger compaction if there are >= 4 files
  1613. one.level0_file_num_compaction_trigger = 4;
  1614. one.write_buffer_size = 120000;
  1615. Reopen({default_cf, one});
  1616. // make sure all background compaction jobs can be scheduled
  1617. auto stop_token =
  1618. dbfull()->TEST_write_controler().GetCompactionPressureToken();
  1619. // SETUP column family "one" -- universal style
  1620. for (int i = 0; i < one.level0_file_num_compaction_trigger - 2; ++i) {
  1621. PutRandomData(1, 10, 12000, true);
  1622. PutRandomData(1, 1, 10, true);
  1623. WaitForFlush(1);
  1624. AssertFilesPerLevel(ToString(i + 1), 1);
  1625. }
  1626. bool cf_1_1 = true;
  1627. bool cf_1_2 = true;
  1628. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  1629. {{"ColumnFamilyTest::ManualAuto:4", "ColumnFamilyTest::ManualAuto:2"},
  1630. {"ColumnFamilyTest::ManualAuto:4", "ColumnFamilyTest::ManualAuto:5"},
  1631. {"ColumnFamilyTest::ManualAuto:1", "ColumnFamilyTest::ManualAuto:2"},
  1632. {"ColumnFamilyTest::ManualAuto:1", "ColumnFamilyTest::ManualAuto:3"}});
  1633. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  1634. "DBImpl::BackgroundCompaction:NonTrivial:AfterRun", [&](void* /*arg*/) {
  1635. if (cf_1_1) {
  1636. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:4");
  1637. cf_1_1 = false;
  1638. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:3");
  1639. } else if (cf_1_2) {
  1640. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:2");
  1641. cf_1_2 = false;
  1642. }
  1643. });
  1644. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  1645. ROCKSDB_NAMESPACE::port::Thread threads([&] {
  1646. CompactRangeOptions compact_options;
  1647. compact_options.exclusive_manual_compaction = false;
  1648. ASSERT_OK(
  1649. db_->CompactRange(compact_options, handles_[1], nullptr, nullptr));
  1650. });
  1651. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:5");
  1652. WaitForFlush(1);
  1653. // Add more L0 files and force automatic compaction
  1654. for (int i = 0; i < one.level0_file_num_compaction_trigger; ++i) {
  1655. PutRandomData(1, 10, 12000, true);
  1656. PutRandomData(1, 1, 10, true);
  1657. WaitForFlush(1);
  1658. AssertFilesPerLevel(ToString(one.level0_file_num_compaction_trigger + i),
  1659. 1);
  1660. }
  1661. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:1");
  1662. threads.join();
  1663. WaitForCompaction();
  1664. // VERIFY compaction "one"
  1665. ASSERT_LE(NumTableFilesAtLevel(0, 1), 2);
  1666. // Compare against saved keys
  1667. std::set<std::string>::iterator key_iter = keys_[1].begin();
  1668. while (key_iter != keys_[1].end()) {
  1669. ASSERT_NE("NOT_FOUND", Get(1, *key_iter));
  1670. key_iter++;
  1671. }
  1672. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  1673. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  1674. }
  1675. TEST_P(ColumnFamilyTest, SameCFManualAutomaticCompactionsLevel) {
  1676. Open();
  1677. CreateColumnFamilies({"one"});
  1678. ColumnFamilyOptions default_cf, one;
  1679. db_options_.max_open_files = 20; // only 10 files in file cache
  1680. db_options_.max_background_compactions = 3;
  1681. default_cf.compaction_style = kCompactionStyleLevel;
  1682. default_cf.num_levels = 3;
  1683. default_cf.write_buffer_size = 64 << 10; // 64KB
  1684. default_cf.target_file_size_base = 30 << 10;
  1685. default_cf.max_compaction_bytes = default_cf.target_file_size_base * 1100;
  1686. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1687. ;
  1688. table_options.no_block_cache = true;
  1689. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1690. one.compaction_style = kCompactionStyleLevel;
  1691. one.num_levels = 1;
  1692. // trigger compaction if there are >= 4 files
  1693. one.level0_file_num_compaction_trigger = 3;
  1694. one.write_buffer_size = 120000;
  1695. Reopen({default_cf, one});
  1696. // make sure all background compaction jobs can be scheduled
  1697. auto stop_token =
  1698. dbfull()->TEST_write_controler().GetCompactionPressureToken();
  1699. // SETUP column family "one" -- level style
  1700. for (int i = 0; i < one.level0_file_num_compaction_trigger - 2; ++i) {
  1701. PutRandomData(1, 10, 12000, true);
  1702. PutRandomData(1, 1, 10, true);
  1703. WaitForFlush(1);
  1704. AssertFilesPerLevel(ToString(i + 1), 1);
  1705. }
  1706. bool cf_1_1 = true;
  1707. bool cf_1_2 = true;
  1708. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  1709. {{"ColumnFamilyTest::ManualAuto:4", "ColumnFamilyTest::ManualAuto:2"},
  1710. {"ColumnFamilyTest::ManualAuto:4", "ColumnFamilyTest::ManualAuto:5"},
  1711. {"ColumnFamilyTest::ManualAuto:3", "ColumnFamilyTest::ManualAuto:2"},
  1712. {"LevelCompactionPicker::PickCompactionBySize:0",
  1713. "ColumnFamilyTest::ManualAuto:3"},
  1714. {"ColumnFamilyTest::ManualAuto:1", "ColumnFamilyTest::ManualAuto:3"}});
  1715. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  1716. "DBImpl::BackgroundCompaction:NonTrivial:AfterRun", [&](void* /*arg*/) {
  1717. if (cf_1_1) {
  1718. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:4");
  1719. cf_1_1 = false;
  1720. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:3");
  1721. } else if (cf_1_2) {
  1722. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:2");
  1723. cf_1_2 = false;
  1724. }
  1725. });
  1726. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  1727. ROCKSDB_NAMESPACE::port::Thread threads([&] {
  1728. CompactRangeOptions compact_options;
  1729. compact_options.exclusive_manual_compaction = false;
  1730. ASSERT_OK(
  1731. db_->CompactRange(compact_options, handles_[1], nullptr, nullptr));
  1732. });
  1733. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:5");
  1734. // Add more L0 files and force automatic compaction
  1735. for (int i = 0; i < one.level0_file_num_compaction_trigger; ++i) {
  1736. PutRandomData(1, 10, 12000, true);
  1737. PutRandomData(1, 1, 10, true);
  1738. WaitForFlush(1);
  1739. AssertFilesPerLevel(ToString(one.level0_file_num_compaction_trigger + i),
  1740. 1);
  1741. }
  1742. TEST_SYNC_POINT("ColumnFamilyTest::ManualAuto:1");
  1743. threads.join();
  1744. WaitForCompaction();
  1745. // VERIFY compaction "one"
  1746. AssertFilesPerLevel("0,1", 1);
  1747. // Compare against saved keys
  1748. std::set<std::string>::iterator key_iter = keys_[1].begin();
  1749. while (key_iter != keys_[1].end()) {
  1750. ASSERT_NE("NOT_FOUND", Get(1, *key_iter));
  1751. key_iter++;
  1752. }
  1753. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  1754. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  1755. }
  1756. // In this test, we generate enough files to trigger automatic compactions.
  1757. // The automatic compaction waits in NonTrivial:AfterRun
  1758. // We generate more files and then trigger an automatic compaction
  1759. // This will wait because the automatic compaction has files it needs.
  1760. // Once the conflict is hit, the automatic compaction starts and ends
  1761. // Then the manual will run and end.
  1762. TEST_P(ColumnFamilyTest, SameCFAutomaticManualCompactions) {
  1763. Open();
  1764. CreateColumnFamilies({"one"});
  1765. ColumnFamilyOptions default_cf, one;
  1766. db_options_.max_open_files = 20; // only 10 files in file cache
  1767. db_options_.max_background_compactions = 3;
  1768. default_cf.compaction_style = kCompactionStyleLevel;
  1769. default_cf.num_levels = 3;
  1770. default_cf.write_buffer_size = 64 << 10; // 64KB
  1771. default_cf.target_file_size_base = 30 << 10;
  1772. default_cf.max_compaction_bytes = default_cf.target_file_size_base * 1100;
  1773. BlockBasedTableOptions table_options = GetBlockBasedTableOptions();
  1774. ;
  1775. table_options.no_block_cache = true;
  1776. default_cf.table_factory.reset(NewBlockBasedTableFactory(table_options));
  1777. one.compaction_style = kCompactionStyleUniversal;
  1778. one.num_levels = 1;
  1779. // trigger compaction if there are >= 4 files
  1780. one.level0_file_num_compaction_trigger = 4;
  1781. one.write_buffer_size = 120000;
  1782. Reopen({default_cf, one});
  1783. // make sure all background compaction jobs can be scheduled
  1784. auto stop_token =
  1785. dbfull()->TEST_write_controler().GetCompactionPressureToken();
  1786. bool cf_1_1 = true;
  1787. bool cf_1_2 = true;
  1788. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  1789. {{"ColumnFamilyTest::AutoManual:4", "ColumnFamilyTest::AutoManual:2"},
  1790. {"ColumnFamilyTest::AutoManual:4", "ColumnFamilyTest::AutoManual:5"},
  1791. {"CompactionPicker::CompactRange:Conflict",
  1792. "ColumnFamilyTest::AutoManual:3"}});
  1793. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  1794. "DBImpl::BackgroundCompaction:NonTrivial:AfterRun", [&](void* /*arg*/) {
  1795. if (cf_1_1) {
  1796. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:4");
  1797. cf_1_1 = false;
  1798. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:3");
  1799. } else if (cf_1_2) {
  1800. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:2");
  1801. cf_1_2 = false;
  1802. }
  1803. });
  1804. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  1805. // SETUP column family "one" -- universal style
  1806. for (int i = 0; i < one.level0_file_num_compaction_trigger; ++i) {
  1807. PutRandomData(1, 10, 12000, true);
  1808. PutRandomData(1, 1, 10, true);
  1809. WaitForFlush(1);
  1810. AssertFilesPerLevel(ToString(i + 1), 1);
  1811. }
  1812. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:5");
  1813. // Add another L0 file and force automatic compaction
  1814. for (int i = 0; i < one.level0_file_num_compaction_trigger - 2; ++i) {
  1815. PutRandomData(1, 10, 12000, true);
  1816. PutRandomData(1, 1, 10, true);
  1817. WaitForFlush(1);
  1818. }
  1819. CompactRangeOptions compact_options;
  1820. compact_options.exclusive_manual_compaction = false;
  1821. ASSERT_OK(db_->CompactRange(compact_options, handles_[1], nullptr, nullptr));
  1822. TEST_SYNC_POINT("ColumnFamilyTest::AutoManual:1");
  1823. WaitForCompaction();
  1824. // VERIFY compaction "one"
  1825. AssertFilesPerLevel("1", 1);
  1826. // Compare against saved keys
  1827. std::set<std::string>::iterator key_iter = keys_[1].begin();
  1828. while (key_iter != keys_[1].end()) {
  1829. ASSERT_NE("NOT_FOUND", Get(1, *key_iter));
  1830. key_iter++;
  1831. }
  1832. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  1833. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  1834. }
  1835. #endif // !ROCKSDB_LITE
  1836. #ifndef ROCKSDB_LITE // Tailing iterator not supported
  1837. namespace {
  1838. std::string IterStatus(Iterator* iter) {
  1839. std::string result;
  1840. if (iter->Valid()) {
  1841. result = iter->key().ToString() + "->" + iter->value().ToString();
  1842. } else {
  1843. result = "(invalid)";
  1844. }
  1845. return result;
  1846. }
  1847. } // anonymous namespace
  1848. TEST_P(ColumnFamilyTest, NewIteratorsTest) {
  1849. // iter == 0 -- no tailing
  1850. // iter == 2 -- tailing
  1851. for (int iter = 0; iter < 2; ++iter) {
  1852. Open();
  1853. CreateColumnFamiliesAndReopen({"one", "two"});
  1854. ASSERT_OK(Put(0, "a", "b"));
  1855. ASSERT_OK(Put(1, "b", "a"));
  1856. ASSERT_OK(Put(2, "c", "m"));
  1857. ASSERT_OK(Put(2, "v", "t"));
  1858. std::vector<Iterator*> iterators;
  1859. ReadOptions options;
  1860. options.tailing = (iter == 1);
  1861. ASSERT_OK(db_->NewIterators(options, handles_, &iterators));
  1862. for (auto it : iterators) {
  1863. it->SeekToFirst();
  1864. }
  1865. ASSERT_EQ(IterStatus(iterators[0]), "a->b");
  1866. ASSERT_EQ(IterStatus(iterators[1]), "b->a");
  1867. ASSERT_EQ(IterStatus(iterators[2]), "c->m");
  1868. ASSERT_OK(Put(1, "x", "x"));
  1869. for (auto it : iterators) {
  1870. it->Next();
  1871. }
  1872. ASSERT_EQ(IterStatus(iterators[0]), "(invalid)");
  1873. if (iter == 0) {
  1874. // no tailing
  1875. ASSERT_EQ(IterStatus(iterators[1]), "(invalid)");
  1876. } else {
  1877. // tailing
  1878. ASSERT_EQ(IterStatus(iterators[1]), "x->x");
  1879. }
  1880. ASSERT_EQ(IterStatus(iterators[2]), "v->t");
  1881. for (auto it : iterators) {
  1882. delete it;
  1883. }
  1884. Destroy();
  1885. }
  1886. }
  1887. #endif // !ROCKSDB_LITE
  1888. #ifndef ROCKSDB_LITE // ReadOnlyDB is not supported
  1889. TEST_P(ColumnFamilyTest, ReadOnlyDBTest) {
  1890. Open();
  1891. CreateColumnFamiliesAndReopen({"one", "two", "three", "four"});
  1892. ASSERT_OK(Put(0, "a", "b"));
  1893. ASSERT_OK(Put(1, "foo", "bla"));
  1894. ASSERT_OK(Put(2, "foo", "blabla"));
  1895. ASSERT_OK(Put(3, "foo", "blablabla"));
  1896. ASSERT_OK(Put(4, "foo", "blablablabla"));
  1897. DropColumnFamilies({2});
  1898. Close();
  1899. // open only a subset of column families
  1900. AssertOpenReadOnly({"default", "one", "four"});
  1901. ASSERT_EQ("NOT_FOUND", Get(0, "foo"));
  1902. ASSERT_EQ("bla", Get(1, "foo"));
  1903. ASSERT_EQ("blablablabla", Get(2, "foo"));
  1904. // test newiterators
  1905. {
  1906. std::vector<Iterator*> iterators;
  1907. ASSERT_OK(db_->NewIterators(ReadOptions(), handles_, &iterators));
  1908. for (auto it : iterators) {
  1909. it->SeekToFirst();
  1910. }
  1911. ASSERT_EQ(IterStatus(iterators[0]), "a->b");
  1912. ASSERT_EQ(IterStatus(iterators[1]), "foo->bla");
  1913. ASSERT_EQ(IterStatus(iterators[2]), "foo->blablablabla");
  1914. for (auto it : iterators) {
  1915. it->Next();
  1916. }
  1917. ASSERT_EQ(IterStatus(iterators[0]), "(invalid)");
  1918. ASSERT_EQ(IterStatus(iterators[1]), "(invalid)");
  1919. ASSERT_EQ(IterStatus(iterators[2]), "(invalid)");
  1920. for (auto it : iterators) {
  1921. delete it;
  1922. }
  1923. }
  1924. Close();
  1925. // can't open dropped column family
  1926. Status s = OpenReadOnly({"default", "one", "two"});
  1927. ASSERT_TRUE(!s.ok());
  1928. // Can't open without specifying default column family
  1929. s = OpenReadOnly({"one", "four"});
  1930. ASSERT_TRUE(!s.ok());
  1931. }
  1932. #endif // !ROCKSDB_LITE
  1933. #ifndef ROCKSDB_LITE // WaitForFlush() is not supported in lite
  1934. TEST_P(ColumnFamilyTest, DontRollEmptyLogs) {
  1935. Open();
  1936. CreateColumnFamiliesAndReopen({"one", "two", "three", "four"});
  1937. for (size_t i = 0; i < handles_.size(); ++i) {
  1938. PutRandomData(static_cast<int>(i), 10, 100);
  1939. }
  1940. int num_writable_file_start = env_->GetNumberOfNewWritableFileCalls();
  1941. // this will trigger the flushes
  1942. for (int i = 0; i <= 4; ++i) {
  1943. ASSERT_OK(Flush(i));
  1944. }
  1945. for (int i = 0; i < 4; ++i) {
  1946. WaitForFlush(i);
  1947. }
  1948. int total_new_writable_files =
  1949. env_->GetNumberOfNewWritableFileCalls() - num_writable_file_start;
  1950. ASSERT_EQ(static_cast<size_t>(total_new_writable_files), handles_.size() + 1);
  1951. Close();
  1952. }
  1953. #endif // !ROCKSDB_LITE
  1954. #ifndef ROCKSDB_LITE // WaitForCompaction() is not supported in lite
  1955. TEST_P(ColumnFamilyTest, FlushStaleColumnFamilies) {
  1956. Open();
  1957. CreateColumnFamilies({"one", "two"});
  1958. ColumnFamilyOptions default_cf, one, two;
  1959. default_cf.write_buffer_size = 100000; // small write buffer size
  1960. default_cf.arena_block_size = 4096;
  1961. default_cf.disable_auto_compactions = true;
  1962. one.disable_auto_compactions = true;
  1963. two.disable_auto_compactions = true;
  1964. db_options_.max_total_wal_size = 210000;
  1965. Reopen({default_cf, one, two});
  1966. PutRandomData(2, 1, 10); // 10 bytes
  1967. for (int i = 0; i < 2; ++i) {
  1968. PutRandomData(0, 100, 1000); // flush
  1969. WaitForFlush(0);
  1970. AssertCountLiveFiles(i + 1);
  1971. }
  1972. // third flush. now, CF [two] should be detected as stale and flushed
  1973. // column family 1 should not be flushed since it's empty
  1974. PutRandomData(0, 100, 1000); // flush
  1975. WaitForFlush(0);
  1976. WaitForFlush(2);
  1977. // 3 files for default column families, 1 file for column family [two], zero
  1978. // files for column family [one], because it's empty
  1979. AssertCountLiveFiles(4);
  1980. Flush(0);
  1981. ASSERT_EQ(0, dbfull()->TEST_total_log_size());
  1982. Close();
  1983. }
  1984. #endif // !ROCKSDB_LITE
  1985. TEST_P(ColumnFamilyTest, CreateMissingColumnFamilies) {
  1986. Status s = TryOpen({"one", "two"});
  1987. ASSERT_TRUE(!s.ok());
  1988. db_options_.create_missing_column_families = true;
  1989. s = TryOpen({"default", "one", "two"});
  1990. ASSERT_TRUE(s.ok());
  1991. Close();
  1992. }
  1993. TEST_P(ColumnFamilyTest, SanitizeOptions) {
  1994. DBOptions db_options;
  1995. for (int s = kCompactionStyleLevel; s <= kCompactionStyleUniversal; ++s) {
  1996. for (int l = 0; l <= 2; l++) {
  1997. for (int i = 1; i <= 3; i++) {
  1998. for (int j = 1; j <= 3; j++) {
  1999. for (int k = 1; k <= 3; k++) {
  2000. ColumnFamilyOptions original;
  2001. original.compaction_style = static_cast<CompactionStyle>(s);
  2002. original.num_levels = l;
  2003. original.level0_stop_writes_trigger = i;
  2004. original.level0_slowdown_writes_trigger = j;
  2005. original.level0_file_num_compaction_trigger = k;
  2006. original.write_buffer_size =
  2007. l * 4 * 1024 * 1024 + i * 1024 * 1024 + j * 1024 + k;
  2008. ColumnFamilyOptions result =
  2009. SanitizeOptions(ImmutableDBOptions(db_options), original);
  2010. ASSERT_TRUE(result.level0_stop_writes_trigger >=
  2011. result.level0_slowdown_writes_trigger);
  2012. ASSERT_TRUE(result.level0_slowdown_writes_trigger >=
  2013. result.level0_file_num_compaction_trigger);
  2014. ASSERT_TRUE(result.level0_file_num_compaction_trigger ==
  2015. original.level0_file_num_compaction_trigger);
  2016. if (s == kCompactionStyleLevel) {
  2017. ASSERT_GE(result.num_levels, 2);
  2018. } else {
  2019. ASSERT_GE(result.num_levels, 1);
  2020. if (original.num_levels >= 1) {
  2021. ASSERT_EQ(result.num_levels, original.num_levels);
  2022. }
  2023. }
  2024. // Make sure Sanitize options sets arena_block_size to 1/8 of
  2025. // the write_buffer_size, rounded up to a multiple of 4k.
  2026. size_t expected_arena_block_size =
  2027. l * 4 * 1024 * 1024 / 8 + i * 1024 * 1024 / 8;
  2028. if (j + k != 0) {
  2029. // not a multiple of 4k, round up 4k
  2030. expected_arena_block_size += 4 * 1024;
  2031. }
  2032. ASSERT_EQ(expected_arena_block_size, result.arena_block_size);
  2033. }
  2034. }
  2035. }
  2036. }
  2037. }
  2038. }
  2039. TEST_P(ColumnFamilyTest, ReadDroppedColumnFamily) {
  2040. // iter 0 -- drop CF, don't reopen
  2041. // iter 1 -- delete CF, reopen
  2042. for (int iter = 0; iter < 2; ++iter) {
  2043. db_options_.create_missing_column_families = true;
  2044. db_options_.max_open_files = 20;
  2045. // delete obsolete files always
  2046. db_options_.delete_obsolete_files_period_micros = 0;
  2047. Open({"default", "one", "two"});
  2048. ColumnFamilyOptions options;
  2049. options.level0_file_num_compaction_trigger = 100;
  2050. options.level0_slowdown_writes_trigger = 200;
  2051. options.level0_stop_writes_trigger = 200;
  2052. options.write_buffer_size = 100000; // small write buffer size
  2053. Reopen({options, options, options});
  2054. // 1MB should create ~10 files for each CF
  2055. int kKeysNum = 10000;
  2056. PutRandomData(0, kKeysNum, 100);
  2057. PutRandomData(1, kKeysNum, 100);
  2058. PutRandomData(2, kKeysNum, 100);
  2059. {
  2060. std::unique_ptr<Iterator> iterator(
  2061. db_->NewIterator(ReadOptions(), handles_[2]));
  2062. iterator->SeekToFirst();
  2063. if (iter == 0) {
  2064. // Drop CF two
  2065. ASSERT_OK(db_->DropColumnFamily(handles_[2]));
  2066. } else {
  2067. // delete CF two
  2068. db_->DestroyColumnFamilyHandle(handles_[2]);
  2069. handles_[2] = nullptr;
  2070. }
  2071. // Make sure iterator created can still be used.
  2072. int count = 0;
  2073. for (; iterator->Valid(); iterator->Next()) {
  2074. ASSERT_OK(iterator->status());
  2075. ++count;
  2076. }
  2077. ASSERT_OK(iterator->status());
  2078. ASSERT_EQ(count, kKeysNum);
  2079. }
  2080. // Add bunch more data to other CFs
  2081. PutRandomData(0, kKeysNum, 100);
  2082. PutRandomData(1, kKeysNum, 100);
  2083. if (iter == 1) {
  2084. Reopen();
  2085. }
  2086. // Since we didn't delete CF handle, RocksDB's contract guarantees that
  2087. // we're still able to read dropped CF
  2088. for (int i = 0; i < 3; ++i) {
  2089. std::unique_ptr<Iterator> iterator(
  2090. db_->NewIterator(ReadOptions(), handles_[i]));
  2091. int count = 0;
  2092. for (iterator->SeekToFirst(); iterator->Valid(); iterator->Next()) {
  2093. ASSERT_OK(iterator->status());
  2094. ++count;
  2095. }
  2096. ASSERT_OK(iterator->status());
  2097. ASSERT_EQ(count, kKeysNum * ((i == 2) ? 1 : 2));
  2098. }
  2099. Close();
  2100. Destroy();
  2101. }
  2102. }
  2103. TEST_P(ColumnFamilyTest, LiveIteratorWithDroppedColumnFamily) {
  2104. db_options_.create_missing_column_families = true;
  2105. db_options_.max_open_files = 20;
  2106. // delete obsolete files always
  2107. db_options_.delete_obsolete_files_period_micros = 0;
  2108. Open({"default", "one", "two"});
  2109. ColumnFamilyOptions options;
  2110. options.level0_file_num_compaction_trigger = 100;
  2111. options.level0_slowdown_writes_trigger = 200;
  2112. options.level0_stop_writes_trigger = 200;
  2113. options.write_buffer_size = 100000; // small write buffer size
  2114. Reopen({options, options, options});
  2115. // 1MB should create ~10 files for each CF
  2116. int kKeysNum = 10000;
  2117. PutRandomData(1, kKeysNum, 100);
  2118. {
  2119. std::unique_ptr<Iterator> iterator(
  2120. db_->NewIterator(ReadOptions(), handles_[1]));
  2121. iterator->SeekToFirst();
  2122. DropColumnFamilies({1});
  2123. // Make sure iterator created can still be used.
  2124. int count = 0;
  2125. for (; iterator->Valid(); iterator->Next()) {
  2126. ASSERT_OK(iterator->status());
  2127. ++count;
  2128. }
  2129. ASSERT_OK(iterator->status());
  2130. ASSERT_EQ(count, kKeysNum);
  2131. }
  2132. Reopen();
  2133. Close();
  2134. Destroy();
  2135. }
  2136. TEST_P(ColumnFamilyTest, FlushAndDropRaceCondition) {
  2137. db_options_.create_missing_column_families = true;
  2138. Open({"default", "one"});
  2139. ColumnFamilyOptions options;
  2140. options.level0_file_num_compaction_trigger = 100;
  2141. options.level0_slowdown_writes_trigger = 200;
  2142. options.level0_stop_writes_trigger = 200;
  2143. options.max_write_buffer_number = 20;
  2144. options.write_buffer_size = 100000; // small write buffer size
  2145. Reopen({options, options});
  2146. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  2147. {{"VersionSet::LogAndApply::ColumnFamilyDrop:0",
  2148. "FlushJob::WriteLevel0Table"},
  2149. {"VersionSet::LogAndApply::ColumnFamilyDrop:1",
  2150. "FlushJob::InstallResults"},
  2151. {"FlushJob::InstallResults",
  2152. "VersionSet::LogAndApply::ColumnFamilyDrop:2"}});
  2153. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  2154. test::SleepingBackgroundTask sleeping_task;
  2155. env_->Schedule(&test::SleepingBackgroundTask::DoSleepTask, &sleeping_task,
  2156. Env::Priority::HIGH);
  2157. // 1MB should create ~10 files for each CF
  2158. int kKeysNum = 10000;
  2159. PutRandomData(1, kKeysNum, 100);
  2160. std::vector<port::Thread> threads;
  2161. threads.emplace_back([&] { ASSERT_OK(db_->DropColumnFamily(handles_[1])); });
  2162. sleeping_task.WakeUp();
  2163. sleeping_task.WaitUntilDone();
  2164. sleeping_task.Reset();
  2165. // now we sleep again. this is just so we're certain that flush job finished
  2166. env_->Schedule(&test::SleepingBackgroundTask::DoSleepTask, &sleeping_task,
  2167. Env::Priority::HIGH);
  2168. sleeping_task.WakeUp();
  2169. sleeping_task.WaitUntilDone();
  2170. {
  2171. // Since we didn't delete CF handle, RocksDB's contract guarantees that
  2172. // we're still able to read dropped CF
  2173. std::unique_ptr<Iterator> iterator(
  2174. db_->NewIterator(ReadOptions(), handles_[1]));
  2175. int count = 0;
  2176. for (iterator->SeekToFirst(); iterator->Valid(); iterator->Next()) {
  2177. ASSERT_OK(iterator->status());
  2178. ++count;
  2179. }
  2180. ASSERT_OK(iterator->status());
  2181. ASSERT_EQ(count, kKeysNum);
  2182. }
  2183. for (auto& t : threads) {
  2184. t.join();
  2185. }
  2186. Close();
  2187. Destroy();
  2188. }
  2189. #ifndef ROCKSDB_LITE
  2190. // skipped as persisting options is not supported in ROCKSDB_LITE
  2191. namespace {
  2192. std::atomic<int> test_stage(0);
  2193. std::atomic<bool> ordered_by_writethread(false);
  2194. const int kMainThreadStartPersistingOptionsFile = 1;
  2195. const int kChildThreadFinishDroppingColumnFamily = 2;
  2196. void DropSingleColumnFamily(ColumnFamilyTest* cf_test, int cf_id,
  2197. std::vector<Comparator*>* comparators) {
  2198. while (test_stage < kMainThreadStartPersistingOptionsFile &&
  2199. !ordered_by_writethread) {
  2200. Env::Default()->SleepForMicroseconds(100);
  2201. }
  2202. cf_test->DropColumnFamilies({cf_id});
  2203. if ((*comparators)[cf_id]) {
  2204. delete (*comparators)[cf_id];
  2205. (*comparators)[cf_id] = nullptr;
  2206. }
  2207. test_stage = kChildThreadFinishDroppingColumnFamily;
  2208. }
  2209. } // namespace
  2210. TEST_P(ColumnFamilyTest, CreateAndDropRace) {
  2211. const int kCfCount = 5;
  2212. std::vector<ColumnFamilyOptions> cf_opts;
  2213. std::vector<Comparator*> comparators;
  2214. for (int i = 0; i < kCfCount; ++i) {
  2215. cf_opts.emplace_back();
  2216. comparators.push_back(new test::SimpleSuffixReverseComparator());
  2217. cf_opts.back().comparator = comparators.back();
  2218. }
  2219. db_options_.create_if_missing = true;
  2220. db_options_.create_missing_column_families = true;
  2221. auto main_thread_id = std::this_thread::get_id();
  2222. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  2223. "PersistRocksDBOptions:start", [&](void* /*arg*/) {
  2224. auto current_thread_id = std::this_thread::get_id();
  2225. // If it's the main thread hitting this sync-point, then it
  2226. // will be blocked until some other thread update the test_stage.
  2227. if (main_thread_id == current_thread_id) {
  2228. test_stage = kMainThreadStartPersistingOptionsFile;
  2229. while (test_stage < kChildThreadFinishDroppingColumnFamily &&
  2230. !ordered_by_writethread) {
  2231. Env::Default()->SleepForMicroseconds(100);
  2232. }
  2233. }
  2234. });
  2235. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->SetCallBack(
  2236. "WriteThread::EnterUnbatched:Wait", [&](void* /*arg*/) {
  2237. // This means a thread doing DropColumnFamily() is waiting for
  2238. // other thread to finish persisting options.
  2239. // In such case, we update the test_stage to unblock the main thread.
  2240. ordered_by_writethread = true;
  2241. });
  2242. // Create a database with four column families
  2243. Open({"default", "one", "two", "three"},
  2244. {cf_opts[0], cf_opts[1], cf_opts[2], cf_opts[3]});
  2245. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  2246. // Start a thread that will drop the first column family
  2247. // and its comparator
  2248. ROCKSDB_NAMESPACE::port::Thread drop_cf_thread(DropSingleColumnFamily, this,
  2249. 1, &comparators);
  2250. DropColumnFamilies({2});
  2251. drop_cf_thread.join();
  2252. Close();
  2253. Destroy();
  2254. for (auto* comparator : comparators) {
  2255. if (comparator) {
  2256. delete comparator;
  2257. }
  2258. }
  2259. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  2260. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->ClearAllCallBacks();
  2261. }
  2262. #endif // !ROCKSDB_LITE
  2263. TEST_P(ColumnFamilyTest, WriteStallSingleColumnFamily) {
  2264. const uint64_t kBaseRate = 800000u;
  2265. db_options_.delayed_write_rate = kBaseRate;
  2266. db_options_.max_background_compactions = 6;
  2267. Open({"default"});
  2268. ColumnFamilyData* cfd =
  2269. static_cast<ColumnFamilyHandleImpl*>(db_->DefaultColumnFamily())->cfd();
  2270. VersionStorageInfo* vstorage = cfd->current()->storage_info();
  2271. MutableCFOptions mutable_cf_options(column_family_options_);
  2272. mutable_cf_options.level0_slowdown_writes_trigger = 20;
  2273. mutable_cf_options.level0_stop_writes_trigger = 10000;
  2274. mutable_cf_options.soft_pending_compaction_bytes_limit = 200;
  2275. mutable_cf_options.hard_pending_compaction_bytes_limit = 2000;
  2276. mutable_cf_options.disable_auto_compactions = false;
  2277. vstorage->TEST_set_estimated_compaction_needed_bytes(50);
  2278. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2279. ASSERT_TRUE(!IsDbWriteStopped());
  2280. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2281. vstorage->TEST_set_estimated_compaction_needed_bytes(201);
  2282. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2283. ASSERT_TRUE(!IsDbWriteStopped());
  2284. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2285. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2286. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2287. vstorage->TEST_set_estimated_compaction_needed_bytes(400);
  2288. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2289. ASSERT_TRUE(!IsDbWriteStopped());
  2290. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2291. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2292. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2293. vstorage->TEST_set_estimated_compaction_needed_bytes(500);
  2294. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2295. ASSERT_TRUE(!IsDbWriteStopped());
  2296. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2297. ASSERT_EQ(kBaseRate / 1.25 / 1.25, GetDbDelayedWriteRate());
  2298. vstorage->TEST_set_estimated_compaction_needed_bytes(450);
  2299. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2300. ASSERT_TRUE(!IsDbWriteStopped());
  2301. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2302. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2303. vstorage->TEST_set_estimated_compaction_needed_bytes(205);
  2304. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2305. ASSERT_TRUE(!IsDbWriteStopped());
  2306. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2307. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2308. vstorage->TEST_set_estimated_compaction_needed_bytes(202);
  2309. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2310. ASSERT_TRUE(!IsDbWriteStopped());
  2311. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2312. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2313. vstorage->TEST_set_estimated_compaction_needed_bytes(201);
  2314. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2315. ASSERT_TRUE(!IsDbWriteStopped());
  2316. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2317. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2318. vstorage->TEST_set_estimated_compaction_needed_bytes(198);
  2319. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2320. ASSERT_TRUE(!IsDbWriteStopped());
  2321. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2322. vstorage->TEST_set_estimated_compaction_needed_bytes(399);
  2323. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2324. ASSERT_TRUE(!IsDbWriteStopped());
  2325. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2326. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2327. vstorage->TEST_set_estimated_compaction_needed_bytes(599);
  2328. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2329. ASSERT_TRUE(!IsDbWriteStopped());
  2330. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2331. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2332. vstorage->TEST_set_estimated_compaction_needed_bytes(2001);
  2333. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2334. ASSERT_TRUE(IsDbWriteStopped());
  2335. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2336. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2337. vstorage->TEST_set_estimated_compaction_needed_bytes(3001);
  2338. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2339. ASSERT_TRUE(IsDbWriteStopped());
  2340. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2341. vstorage->TEST_set_estimated_compaction_needed_bytes(390);
  2342. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2343. ASSERT_TRUE(!IsDbWriteStopped());
  2344. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2345. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2346. vstorage->TEST_set_estimated_compaction_needed_bytes(100);
  2347. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2348. ASSERT_TRUE(!IsDbWriteStopped());
  2349. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2350. vstorage->set_l0_delay_trigger_count(100);
  2351. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2352. ASSERT_TRUE(!IsDbWriteStopped());
  2353. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2354. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2355. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2356. vstorage->set_l0_delay_trigger_count(101);
  2357. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2358. ASSERT_TRUE(!IsDbWriteStopped());
  2359. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2360. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2361. vstorage->set_l0_delay_trigger_count(0);
  2362. vstorage->TEST_set_estimated_compaction_needed_bytes(300);
  2363. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2364. ASSERT_TRUE(!IsDbWriteStopped());
  2365. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2366. ASSERT_EQ(kBaseRate / 1.25 / 1.25, GetDbDelayedWriteRate());
  2367. vstorage->set_l0_delay_trigger_count(101);
  2368. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2369. ASSERT_TRUE(!IsDbWriteStopped());
  2370. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2371. ASSERT_EQ(kBaseRate / 1.25 / 1.25 / 1.25, GetDbDelayedWriteRate());
  2372. vstorage->TEST_set_estimated_compaction_needed_bytes(200);
  2373. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2374. ASSERT_TRUE(!IsDbWriteStopped());
  2375. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2376. ASSERT_EQ(kBaseRate / 1.25 / 1.25, GetDbDelayedWriteRate());
  2377. vstorage->set_l0_delay_trigger_count(0);
  2378. vstorage->TEST_set_estimated_compaction_needed_bytes(0);
  2379. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2380. ASSERT_TRUE(!IsDbWriteStopped());
  2381. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2382. mutable_cf_options.disable_auto_compactions = true;
  2383. dbfull()->TEST_write_controler().set_delayed_write_rate(kBaseRate);
  2384. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2385. ASSERT_TRUE(!IsDbWriteStopped());
  2386. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2387. vstorage->set_l0_delay_trigger_count(50);
  2388. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2389. ASSERT_TRUE(!IsDbWriteStopped());
  2390. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2391. ASSERT_EQ(0, GetDbDelayedWriteRate());
  2392. ASSERT_EQ(kBaseRate, dbfull()->TEST_write_controler().delayed_write_rate());
  2393. vstorage->set_l0_delay_trigger_count(60);
  2394. vstorage->TEST_set_estimated_compaction_needed_bytes(300);
  2395. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2396. ASSERT_TRUE(!IsDbWriteStopped());
  2397. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2398. ASSERT_EQ(0, GetDbDelayedWriteRate());
  2399. ASSERT_EQ(kBaseRate, dbfull()->TEST_write_controler().delayed_write_rate());
  2400. mutable_cf_options.disable_auto_compactions = false;
  2401. vstorage->set_l0_delay_trigger_count(70);
  2402. vstorage->TEST_set_estimated_compaction_needed_bytes(500);
  2403. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2404. ASSERT_TRUE(!IsDbWriteStopped());
  2405. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2406. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2407. vstorage->set_l0_delay_trigger_count(71);
  2408. vstorage->TEST_set_estimated_compaction_needed_bytes(501);
  2409. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2410. ASSERT_TRUE(!IsDbWriteStopped());
  2411. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2412. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2413. }
  2414. TEST_P(ColumnFamilyTest, CompactionSpeedupSingleColumnFamily) {
  2415. db_options_.max_background_compactions = 6;
  2416. Open({"default"});
  2417. ColumnFamilyData* cfd =
  2418. static_cast<ColumnFamilyHandleImpl*>(db_->DefaultColumnFamily())->cfd();
  2419. VersionStorageInfo* vstorage = cfd->current()->storage_info();
  2420. MutableCFOptions mutable_cf_options(column_family_options_);
  2421. // Speed up threshold = min(4 * 2, 4 + (36 - 4)/4) = 8
  2422. mutable_cf_options.level0_file_num_compaction_trigger = 4;
  2423. mutable_cf_options.level0_slowdown_writes_trigger = 36;
  2424. mutable_cf_options.level0_stop_writes_trigger = 50;
  2425. // Speedup threshold = 200 / 4 = 50
  2426. mutable_cf_options.soft_pending_compaction_bytes_limit = 200;
  2427. mutable_cf_options.hard_pending_compaction_bytes_limit = 2000;
  2428. vstorage->TEST_set_estimated_compaction_needed_bytes(40);
  2429. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2430. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2431. vstorage->TEST_set_estimated_compaction_needed_bytes(50);
  2432. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2433. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2434. vstorage->TEST_set_estimated_compaction_needed_bytes(300);
  2435. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2436. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2437. vstorage->TEST_set_estimated_compaction_needed_bytes(45);
  2438. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2439. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2440. vstorage->set_l0_delay_trigger_count(7);
  2441. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2442. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2443. vstorage->set_l0_delay_trigger_count(9);
  2444. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2445. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2446. vstorage->set_l0_delay_trigger_count(6);
  2447. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2448. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2449. // Speed up threshold = min(4 * 2, 4 + (12 - 4)/4) = 6
  2450. mutable_cf_options.level0_file_num_compaction_trigger = 4;
  2451. mutable_cf_options.level0_slowdown_writes_trigger = 16;
  2452. mutable_cf_options.level0_stop_writes_trigger = 30;
  2453. vstorage->set_l0_delay_trigger_count(5);
  2454. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2455. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2456. vstorage->set_l0_delay_trigger_count(7);
  2457. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2458. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2459. vstorage->set_l0_delay_trigger_count(3);
  2460. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2461. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2462. }
  2463. TEST_P(ColumnFamilyTest, WriteStallTwoColumnFamilies) {
  2464. const uint64_t kBaseRate = 810000u;
  2465. db_options_.delayed_write_rate = kBaseRate;
  2466. Open();
  2467. CreateColumnFamilies({"one"});
  2468. ColumnFamilyData* cfd =
  2469. static_cast<ColumnFamilyHandleImpl*>(db_->DefaultColumnFamily())->cfd();
  2470. VersionStorageInfo* vstorage = cfd->current()->storage_info();
  2471. ColumnFamilyData* cfd1 =
  2472. static_cast<ColumnFamilyHandleImpl*>(handles_[1])->cfd();
  2473. VersionStorageInfo* vstorage1 = cfd1->current()->storage_info();
  2474. MutableCFOptions mutable_cf_options(column_family_options_);
  2475. mutable_cf_options.level0_slowdown_writes_trigger = 20;
  2476. mutable_cf_options.level0_stop_writes_trigger = 10000;
  2477. mutable_cf_options.soft_pending_compaction_bytes_limit = 200;
  2478. mutable_cf_options.hard_pending_compaction_bytes_limit = 2000;
  2479. MutableCFOptions mutable_cf_options1 = mutable_cf_options;
  2480. mutable_cf_options1.soft_pending_compaction_bytes_limit = 500;
  2481. vstorage->TEST_set_estimated_compaction_needed_bytes(50);
  2482. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2483. ASSERT_TRUE(!IsDbWriteStopped());
  2484. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2485. vstorage1->TEST_set_estimated_compaction_needed_bytes(201);
  2486. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2487. ASSERT_TRUE(!IsDbWriteStopped());
  2488. ASSERT_TRUE(!dbfull()->TEST_write_controler().NeedsDelay());
  2489. vstorage1->TEST_set_estimated_compaction_needed_bytes(600);
  2490. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2491. ASSERT_TRUE(!IsDbWriteStopped());
  2492. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2493. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2494. vstorage->TEST_set_estimated_compaction_needed_bytes(70);
  2495. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2496. ASSERT_TRUE(!IsDbWriteStopped());
  2497. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2498. ASSERT_EQ(kBaseRate, GetDbDelayedWriteRate());
  2499. vstorage1->TEST_set_estimated_compaction_needed_bytes(800);
  2500. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2501. ASSERT_TRUE(!IsDbWriteStopped());
  2502. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2503. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2504. vstorage->TEST_set_estimated_compaction_needed_bytes(300);
  2505. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2506. ASSERT_TRUE(!IsDbWriteStopped());
  2507. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2508. ASSERT_EQ(kBaseRate / 1.25 / 1.25, GetDbDelayedWriteRate());
  2509. vstorage1->TEST_set_estimated_compaction_needed_bytes(700);
  2510. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2511. ASSERT_TRUE(!IsDbWriteStopped());
  2512. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2513. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2514. vstorage->TEST_set_estimated_compaction_needed_bytes(500);
  2515. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2516. ASSERT_TRUE(!IsDbWriteStopped());
  2517. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2518. ASSERT_EQ(kBaseRate / 1.25 / 1.25, GetDbDelayedWriteRate());
  2519. vstorage1->TEST_set_estimated_compaction_needed_bytes(600);
  2520. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2521. ASSERT_TRUE(!IsDbWriteStopped());
  2522. ASSERT_TRUE(dbfull()->TEST_write_controler().NeedsDelay());
  2523. ASSERT_EQ(kBaseRate / 1.25, GetDbDelayedWriteRate());
  2524. }
  2525. TEST_P(ColumnFamilyTest, CompactionSpeedupTwoColumnFamilies) {
  2526. db_options_.max_background_compactions = 6;
  2527. column_family_options_.soft_pending_compaction_bytes_limit = 200;
  2528. column_family_options_.hard_pending_compaction_bytes_limit = 2000;
  2529. Open();
  2530. CreateColumnFamilies({"one"});
  2531. ColumnFamilyData* cfd =
  2532. static_cast<ColumnFamilyHandleImpl*>(db_->DefaultColumnFamily())->cfd();
  2533. VersionStorageInfo* vstorage = cfd->current()->storage_info();
  2534. ColumnFamilyData* cfd1 =
  2535. static_cast<ColumnFamilyHandleImpl*>(handles_[1])->cfd();
  2536. VersionStorageInfo* vstorage1 = cfd1->current()->storage_info();
  2537. MutableCFOptions mutable_cf_options(column_family_options_);
  2538. // Speed up threshold = min(4 * 2, 4 + (36 - 4)/4) = 8
  2539. mutable_cf_options.level0_file_num_compaction_trigger = 4;
  2540. mutable_cf_options.level0_slowdown_writes_trigger = 36;
  2541. mutable_cf_options.level0_stop_writes_trigger = 30;
  2542. // Speedup threshold = 200 / 4 = 50
  2543. mutable_cf_options.soft_pending_compaction_bytes_limit = 200;
  2544. mutable_cf_options.hard_pending_compaction_bytes_limit = 2000;
  2545. MutableCFOptions mutable_cf_options1 = mutable_cf_options;
  2546. mutable_cf_options1.level0_slowdown_writes_trigger = 16;
  2547. vstorage->TEST_set_estimated_compaction_needed_bytes(40);
  2548. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2549. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2550. vstorage->TEST_set_estimated_compaction_needed_bytes(60);
  2551. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2552. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2553. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2554. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2555. vstorage1->TEST_set_estimated_compaction_needed_bytes(30);
  2556. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2557. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2558. vstorage1->TEST_set_estimated_compaction_needed_bytes(70);
  2559. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2560. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2561. vstorage->TEST_set_estimated_compaction_needed_bytes(20);
  2562. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2563. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2564. vstorage1->TEST_set_estimated_compaction_needed_bytes(3);
  2565. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2566. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2567. vstorage->set_l0_delay_trigger_count(9);
  2568. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2569. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2570. vstorage1->set_l0_delay_trigger_count(2);
  2571. RecalculateWriteStallConditions(cfd1, mutable_cf_options);
  2572. ASSERT_EQ(6, dbfull()->TEST_BGCompactionsAllowed());
  2573. vstorage->set_l0_delay_trigger_count(0);
  2574. RecalculateWriteStallConditions(cfd, mutable_cf_options);
  2575. ASSERT_EQ(1, dbfull()->TEST_BGCompactionsAllowed());
  2576. }
  2577. TEST_P(ColumnFamilyTest, CreateAndDestoryOptions) {
  2578. std::unique_ptr<ColumnFamilyOptions> cfo(new ColumnFamilyOptions());
  2579. ColumnFamilyHandle* cfh;
  2580. Open();
  2581. ASSERT_OK(db_->CreateColumnFamily(*(cfo.get()), "yoyo", &cfh));
  2582. cfo.reset();
  2583. ASSERT_OK(db_->Put(WriteOptions(), cfh, "foo", "bar"));
  2584. ASSERT_OK(db_->Flush(FlushOptions(), cfh));
  2585. ASSERT_OK(db_->DropColumnFamily(cfh));
  2586. ASSERT_OK(db_->DestroyColumnFamilyHandle(cfh));
  2587. }
  2588. TEST_P(ColumnFamilyTest, CreateDropAndDestroy) {
  2589. ColumnFamilyHandle* cfh;
  2590. Open();
  2591. ASSERT_OK(db_->CreateColumnFamily(ColumnFamilyOptions(), "yoyo", &cfh));
  2592. ASSERT_OK(db_->Put(WriteOptions(), cfh, "foo", "bar"));
  2593. ASSERT_OK(db_->Flush(FlushOptions(), cfh));
  2594. ASSERT_OK(db_->DropColumnFamily(cfh));
  2595. ASSERT_OK(db_->DestroyColumnFamilyHandle(cfh));
  2596. }
  2597. #ifndef ROCKSDB_LITE
  2598. TEST_P(ColumnFamilyTest, CreateDropAndDestroyWithoutFileDeletion) {
  2599. ColumnFamilyHandle* cfh;
  2600. Open();
  2601. ASSERT_OK(db_->CreateColumnFamily(ColumnFamilyOptions(), "yoyo", &cfh));
  2602. ASSERT_OK(db_->Put(WriteOptions(), cfh, "foo", "bar"));
  2603. ASSERT_OK(db_->Flush(FlushOptions(), cfh));
  2604. ASSERT_OK(db_->DisableFileDeletions());
  2605. ASSERT_OK(db_->DropColumnFamily(cfh));
  2606. ASSERT_OK(db_->DestroyColumnFamilyHandle(cfh));
  2607. }
  2608. TEST_P(ColumnFamilyTest, FlushCloseWALFiles) {
  2609. SpecialEnv env(Env::Default());
  2610. db_options_.env = &env;
  2611. db_options_.max_background_flushes = 1;
  2612. column_family_options_.memtable_factory.reset(new SpecialSkipListFactory(2));
  2613. Open();
  2614. CreateColumnFamilies({"one"});
  2615. ASSERT_OK(Put(1, "fodor", "mirko"));
  2616. ASSERT_OK(Put(0, "fodor", "mirko"));
  2617. ASSERT_OK(Put(1, "fodor", "mirko"));
  2618. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency({
  2619. {"DBImpl::BGWorkFlush:done", "FlushCloseWALFiles:0"},
  2620. });
  2621. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  2622. // Block flush jobs from running
  2623. test::SleepingBackgroundTask sleeping_task;
  2624. env_->Schedule(&test::SleepingBackgroundTask::DoSleepTask, &sleeping_task,
  2625. Env::Priority::HIGH);
  2626. WriteOptions wo;
  2627. wo.sync = true;
  2628. ASSERT_OK(db_->Put(wo, handles_[1], "fodor", "mirko"));
  2629. ASSERT_EQ(2, env.num_open_wal_file_.load());
  2630. sleeping_task.WakeUp();
  2631. sleeping_task.WaitUntilDone();
  2632. TEST_SYNC_POINT("FlushCloseWALFiles:0");
  2633. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  2634. ASSERT_EQ(1, env.num_open_wal_file_.load());
  2635. Reopen();
  2636. ASSERT_EQ("mirko", Get(0, "fodor"));
  2637. ASSERT_EQ("mirko", Get(1, "fodor"));
  2638. db_options_.env = env_;
  2639. Close();
  2640. }
  2641. #endif // !ROCKSDB_LITE
  2642. #ifndef ROCKSDB_LITE // WaitForFlush() is not supported
  2643. TEST_P(ColumnFamilyTest, IteratorCloseWALFile1) {
  2644. SpecialEnv env(Env::Default());
  2645. db_options_.env = &env;
  2646. db_options_.max_background_flushes = 1;
  2647. column_family_options_.memtable_factory.reset(new SpecialSkipListFactory(2));
  2648. Open();
  2649. CreateColumnFamilies({"one"});
  2650. ASSERT_OK(Put(1, "fodor", "mirko"));
  2651. // Create an iterator holding the current super version.
  2652. Iterator* it = db_->NewIterator(ReadOptions(), handles_[1]);
  2653. // A flush will make `it` hold the last reference of its super version.
  2654. Flush(1);
  2655. ASSERT_OK(Put(1, "fodor", "mirko"));
  2656. ASSERT_OK(Put(0, "fodor", "mirko"));
  2657. ASSERT_OK(Put(1, "fodor", "mirko"));
  2658. // Flush jobs will close previous WAL files after finishing. By
  2659. // block flush jobs from running, we trigger a condition where
  2660. // the iterator destructor should close the WAL files.
  2661. test::SleepingBackgroundTask sleeping_task;
  2662. env_->Schedule(&test::SleepingBackgroundTask::DoSleepTask, &sleeping_task,
  2663. Env::Priority::HIGH);
  2664. WriteOptions wo;
  2665. wo.sync = true;
  2666. ASSERT_OK(db_->Put(wo, handles_[1], "fodor", "mirko"));
  2667. ASSERT_EQ(2, env.num_open_wal_file_.load());
  2668. // Deleting the iterator will clear its super version, triggering
  2669. // closing all files
  2670. delete it;
  2671. ASSERT_EQ(1, env.num_open_wal_file_.load());
  2672. sleeping_task.WakeUp();
  2673. sleeping_task.WaitUntilDone();
  2674. WaitForFlush(1);
  2675. Reopen();
  2676. ASSERT_EQ("mirko", Get(0, "fodor"));
  2677. ASSERT_EQ("mirko", Get(1, "fodor"));
  2678. db_options_.env = env_;
  2679. Close();
  2680. }
  2681. TEST_P(ColumnFamilyTest, IteratorCloseWALFile2) {
  2682. SpecialEnv env(Env::Default());
  2683. // Allow both of flush and purge job to schedule.
  2684. env.SetBackgroundThreads(2, Env::HIGH);
  2685. db_options_.env = &env;
  2686. db_options_.max_background_flushes = 1;
  2687. column_family_options_.memtable_factory.reset(new SpecialSkipListFactory(2));
  2688. Open();
  2689. CreateColumnFamilies({"one"});
  2690. ASSERT_OK(Put(1, "fodor", "mirko"));
  2691. // Create an iterator holding the current super version.
  2692. ReadOptions ro;
  2693. ro.background_purge_on_iterator_cleanup = true;
  2694. Iterator* it = db_->NewIterator(ro, handles_[1]);
  2695. // A flush will make `it` hold the last reference of its super version.
  2696. Flush(1);
  2697. ASSERT_OK(Put(1, "fodor", "mirko"));
  2698. ASSERT_OK(Put(0, "fodor", "mirko"));
  2699. ASSERT_OK(Put(1, "fodor", "mirko"));
  2700. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency({
  2701. {"ColumnFamilyTest::IteratorCloseWALFile2:0",
  2702. "DBImpl::BGWorkPurge:start"},
  2703. {"ColumnFamilyTest::IteratorCloseWALFile2:2",
  2704. "DBImpl::BackgroundCallFlush:start"},
  2705. {"DBImpl::BGWorkPurge:end", "ColumnFamilyTest::IteratorCloseWALFile2:1"},
  2706. });
  2707. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  2708. WriteOptions wo;
  2709. wo.sync = true;
  2710. ASSERT_OK(db_->Put(wo, handles_[1], "fodor", "mirko"));
  2711. ASSERT_EQ(2, env.num_open_wal_file_.load());
  2712. // Deleting the iterator will clear its super version, triggering
  2713. // closing all files
  2714. delete it;
  2715. ASSERT_EQ(2, env.num_open_wal_file_.load());
  2716. TEST_SYNC_POINT("ColumnFamilyTest::IteratorCloseWALFile2:0");
  2717. TEST_SYNC_POINT("ColumnFamilyTest::IteratorCloseWALFile2:1");
  2718. ASSERT_EQ(1, env.num_open_wal_file_.load());
  2719. TEST_SYNC_POINT("ColumnFamilyTest::IteratorCloseWALFile2:2");
  2720. WaitForFlush(1);
  2721. ASSERT_EQ(1, env.num_open_wal_file_.load());
  2722. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  2723. Reopen();
  2724. ASSERT_EQ("mirko", Get(0, "fodor"));
  2725. ASSERT_EQ("mirko", Get(1, "fodor"));
  2726. db_options_.env = env_;
  2727. Close();
  2728. }
  2729. #endif // !ROCKSDB_LITE
  2730. #ifndef ROCKSDB_LITE // TEST functions are not supported in lite
  2731. TEST_P(ColumnFamilyTest, ForwardIteratorCloseWALFile) {
  2732. SpecialEnv env(Env::Default());
  2733. // Allow both of flush and purge job to schedule.
  2734. env.SetBackgroundThreads(2, Env::HIGH);
  2735. db_options_.env = &env;
  2736. db_options_.max_background_flushes = 1;
  2737. column_family_options_.memtable_factory.reset(new SpecialSkipListFactory(3));
  2738. column_family_options_.level0_file_num_compaction_trigger = 2;
  2739. Open();
  2740. CreateColumnFamilies({"one"});
  2741. ASSERT_OK(Put(1, "fodor", "mirko"));
  2742. ASSERT_OK(Put(1, "fodar2", "mirko"));
  2743. Flush(1);
  2744. // Create an iterator holding the current super version, as well as
  2745. // the SST file just flushed.
  2746. ReadOptions ro;
  2747. ro.tailing = true;
  2748. ro.background_purge_on_iterator_cleanup = true;
  2749. Iterator* it = db_->NewIterator(ro, handles_[1]);
  2750. // A flush will make `it` hold the last reference of its super version.
  2751. ASSERT_OK(Put(1, "fodor", "mirko"));
  2752. ASSERT_OK(Put(1, "fodar2", "mirko"));
  2753. Flush(1);
  2754. WaitForCompaction();
  2755. ASSERT_OK(Put(1, "fodor", "mirko"));
  2756. ASSERT_OK(Put(1, "fodor", "mirko"));
  2757. ASSERT_OK(Put(0, "fodor", "mirko"));
  2758. ASSERT_OK(Put(1, "fodor", "mirko"));
  2759. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency({
  2760. {"ColumnFamilyTest::IteratorCloseWALFile2:0",
  2761. "DBImpl::BGWorkPurge:start"},
  2762. {"ColumnFamilyTest::IteratorCloseWALFile2:2",
  2763. "DBImpl::BackgroundCallFlush:start"},
  2764. {"DBImpl::BGWorkPurge:end", "ColumnFamilyTest::IteratorCloseWALFile2:1"},
  2765. });
  2766. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  2767. WriteOptions wo;
  2768. wo.sync = true;
  2769. ASSERT_OK(db_->Put(wo, handles_[1], "fodor", "mirko"));
  2770. env.delete_count_.store(0);
  2771. ASSERT_EQ(2, env.num_open_wal_file_.load());
  2772. // Deleting the iterator will clear its super version, triggering
  2773. // closing all files
  2774. it->Seek("");
  2775. ASSERT_EQ(2, env.num_open_wal_file_.load());
  2776. ASSERT_EQ(0, env.delete_count_.load());
  2777. TEST_SYNC_POINT("ColumnFamilyTest::IteratorCloseWALFile2:0");
  2778. TEST_SYNC_POINT("ColumnFamilyTest::IteratorCloseWALFile2:1");
  2779. ASSERT_EQ(1, env.num_open_wal_file_.load());
  2780. ASSERT_EQ(1, env.delete_count_.load());
  2781. TEST_SYNC_POINT("ColumnFamilyTest::IteratorCloseWALFile2:2");
  2782. WaitForFlush(1);
  2783. ASSERT_EQ(1, env.num_open_wal_file_.load());
  2784. ASSERT_EQ(1, env.delete_count_.load());
  2785. delete it;
  2786. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  2787. Reopen();
  2788. ASSERT_EQ("mirko", Get(0, "fodor"));
  2789. ASSERT_EQ("mirko", Get(1, "fodor"));
  2790. db_options_.env = env_;
  2791. Close();
  2792. }
  2793. #endif // !ROCKSDB_LITE
  2794. // Disable on windows because SyncWAL requires env->IsSyncThreadSafe()
  2795. // to return true which is not so in unbuffered mode.
  2796. #ifndef OS_WIN
  2797. TEST_P(ColumnFamilyTest, LogSyncConflictFlush) {
  2798. Open();
  2799. CreateColumnFamiliesAndReopen({"one", "two"});
  2800. Put(0, "", "");
  2801. Put(1, "foo", "bar");
  2802. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->LoadDependency(
  2803. {{"DBImpl::SyncWAL:BeforeMarkLogsSynced:1",
  2804. "ColumnFamilyTest::LogSyncConflictFlush:1"},
  2805. {"ColumnFamilyTest::LogSyncConflictFlush:2",
  2806. "DBImpl::SyncWAL:BeforeMarkLogsSynced:2"}});
  2807. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->EnableProcessing();
  2808. ROCKSDB_NAMESPACE::port::Thread thread([&] { db_->SyncWAL(); });
  2809. TEST_SYNC_POINT("ColumnFamilyTest::LogSyncConflictFlush:1");
  2810. Flush(1);
  2811. Put(1, "foo", "bar");
  2812. Flush(1);
  2813. TEST_SYNC_POINT("ColumnFamilyTest::LogSyncConflictFlush:2");
  2814. thread.join();
  2815. ROCKSDB_NAMESPACE::SyncPoint::GetInstance()->DisableProcessing();
  2816. Close();
  2817. }
  2818. #endif
  2819. // this test is placed here, because the infrastructure for Column Family
  2820. // test is being used to ensure a roll of wal files.
  2821. // Basic idea is to test that WAL truncation is being detected and not
  2822. // ignored
  2823. TEST_P(ColumnFamilyTest, DISABLED_LogTruncationTest) {
  2824. Open();
  2825. CreateColumnFamiliesAndReopen({"one", "two"});
  2826. Build(0, 100);
  2827. // Flush the 0th column family to force a roll of the wal log
  2828. Flush(0);
  2829. // Add some more entries
  2830. Build(100, 100);
  2831. std::vector<std::string> filenames;
  2832. ASSERT_OK(env_->GetChildren(dbname_, &filenames));
  2833. // collect wal files
  2834. std::vector<std::string> logfs;
  2835. for (size_t i = 0; i < filenames.size(); i++) {
  2836. uint64_t number;
  2837. FileType type;
  2838. if (!(ParseFileName(filenames[i], &number, &type))) continue;
  2839. if (type != kLogFile) continue;
  2840. logfs.push_back(filenames[i]);
  2841. }
  2842. std::sort(logfs.begin(), logfs.end());
  2843. ASSERT_GE(logfs.size(), 2);
  2844. // Take the last but one file, and truncate it
  2845. std::string fpath = dbname_ + "/" + logfs[logfs.size() - 2];
  2846. std::vector<std::string> names_save = names_;
  2847. uint64_t fsize;
  2848. ASSERT_OK(env_->GetFileSize(fpath, &fsize));
  2849. ASSERT_GT(fsize, 0);
  2850. Close();
  2851. std::string backup_logs = dbname_ + "/backup_logs";
  2852. std::string t_fpath = backup_logs + "/" + logfs[logfs.size() - 2];
  2853. ASSERT_OK(env_->CreateDirIfMissing(backup_logs));
  2854. // Not sure how easy it is to make this data driven.
  2855. // need to read back the WAL file and truncate last 10
  2856. // entries
  2857. CopyFile(fpath, t_fpath, fsize - 9180);
  2858. ASSERT_OK(env_->DeleteFile(fpath));
  2859. ASSERT_OK(env_->RenameFile(t_fpath, fpath));
  2860. db_options_.wal_recovery_mode = WALRecoveryMode::kPointInTimeRecovery;
  2861. OpenReadOnly(names_save);
  2862. CheckMissed();
  2863. Close();
  2864. Open(names_save);
  2865. CheckMissed();
  2866. Close();
  2867. // cleanup
  2868. env_->DeleteDir(backup_logs);
  2869. }
  2870. TEST_P(ColumnFamilyTest, DefaultCfPathsTest) {
  2871. Open();
  2872. // Leave cf_paths for one column families to be empty.
  2873. // Files should be generated according to db_paths for that
  2874. // column family.
  2875. ColumnFamilyOptions cf_opt1, cf_opt2;
  2876. cf_opt1.cf_paths.emplace_back(dbname_ + "_one_1",
  2877. std::numeric_limits<uint64_t>::max());
  2878. CreateColumnFamilies({"one", "two"}, {cf_opt1, cf_opt2});
  2879. Reopen({ColumnFamilyOptions(), cf_opt1, cf_opt2});
  2880. // Fill Column family 1.
  2881. PutRandomData(1, 100, 100);
  2882. Flush(1);
  2883. ASSERT_EQ(1, GetSstFileCount(cf_opt1.cf_paths[0].path));
  2884. ASSERT_EQ(0, GetSstFileCount(dbname_));
  2885. // Fill column family 2
  2886. PutRandomData(2, 100, 100);
  2887. Flush(2);
  2888. // SST from Column family 2 should be generated in
  2889. // db_paths which is dbname_ in this case.
  2890. ASSERT_EQ(1, GetSstFileCount(dbname_));
  2891. }
  2892. TEST_P(ColumnFamilyTest, MultipleCFPathsTest) {
  2893. Open();
  2894. // Configure Column family specific paths.
  2895. ColumnFamilyOptions cf_opt1, cf_opt2;
  2896. cf_opt1.cf_paths.emplace_back(dbname_ + "_one_1",
  2897. std::numeric_limits<uint64_t>::max());
  2898. cf_opt2.cf_paths.emplace_back(dbname_ + "_two_1",
  2899. std::numeric_limits<uint64_t>::max());
  2900. CreateColumnFamilies({"one", "two"}, {cf_opt1, cf_opt2});
  2901. Reopen({ColumnFamilyOptions(), cf_opt1, cf_opt2});
  2902. PutRandomData(1, 100, 100, true /* save */);
  2903. Flush(1);
  2904. // Check that files are generated in appropriate paths.
  2905. ASSERT_EQ(1, GetSstFileCount(cf_opt1.cf_paths[0].path));
  2906. ASSERT_EQ(0, GetSstFileCount(dbname_));
  2907. PutRandomData(2, 100, 100, true /* save */);
  2908. Flush(2);
  2909. ASSERT_EQ(1, GetSstFileCount(cf_opt2.cf_paths[0].path));
  2910. ASSERT_EQ(0, GetSstFileCount(dbname_));
  2911. // Re-open and verify the keys.
  2912. Reopen({ColumnFamilyOptions(), cf_opt1, cf_opt2});
  2913. DBImpl* dbi = reinterpret_cast<DBImpl*>(db_);
  2914. for (int cf = 1; cf != 3; ++cf) {
  2915. ReadOptions read_options;
  2916. read_options.readahead_size = 0;
  2917. auto it = dbi->NewIterator(read_options, handles_[cf]);
  2918. for (it->SeekToFirst(); it->Valid(); it->Next()) {
  2919. Slice key(it->key());
  2920. ASSERT_NE(keys_[cf].end(), keys_[cf].find(key.ToString()));
  2921. }
  2922. delete it;
  2923. for (const auto& key : keys_[cf]) {
  2924. ASSERT_NE("NOT_FOUND", Get(cf, key));
  2925. }
  2926. }
  2927. }
  2928. } // namespace ROCKSDB_NAMESPACE
  2929. #ifdef ROCKSDB_UNITTESTS_WITH_CUSTOM_OBJECTS_FROM_STATIC_LIBS
  2930. extern "C" {
  2931. void RegisterCustomObjects(int argc, char** argv);
  2932. }
  2933. #else
  2934. void RegisterCustomObjects(int /*argc*/, char** /*argv*/) {}
  2935. #endif // !ROCKSDB_UNITTESTS_WITH_CUSTOM_OBJECTS_FROM_STATIC_LIBS
  2936. int main(int argc, char** argv) {
  2937. ROCKSDB_NAMESPACE::port::InstallStackTraceHandler();
  2938. ::testing::InitGoogleTest(&argc, argv);
  2939. RegisterCustomObjects(argc, argv);
  2940. return RUN_ALL_TESTS();
  2941. }