Iterator support

Add iterator support, used initially only for retrieving bucket
statistics.

The iterator is supported by exporting a function, and when the function
is claled it will take a snapshot of the ledger, run the iterator and
hten close the snapshot.

This required a numbe rof underlying changes, in particular to get key
comparison to work as "expected".  The code had previously misunderstood
how comparison worked between Erlang terms, and in particular did not
account for tuple length being compared first by size of the tuple (and
not just by each element in order).
This commit is contained in:
martinsumner 2016-10-12 17:12:49 +01:00
parent d2cc07a9eb
commit 0a08867280
6 changed files with 762 additions and 202 deletions

View file

@ -71,7 +71,7 @@
%% The Bookie should generate a series of ledger key changes from this
%% information, using a function passed in at startup. For Riak this will be
%% of the form:
%% {{o, Bucket, Key},
%% {{o, Bucket, Key, SubKey|null},
%% SQN,
%% {Hash, Size, {Riak_Metadata}},
%% {active, TS}|{tomb, TS}} or
@ -136,6 +136,7 @@
book_riakput/3,
book_riakget/3,
book_riakhead/3,
book_returnfolder/2,
book_snapshotstore/3,
book_snapshotledger/3,
book_compactjournal/2,
@ -144,7 +145,11 @@
strip_to_keyseqonly/1,
strip_to_seqonly/1,
strip_to_statusonly/1,
striphead_to_details/1]).
strip_to_keyseqstatusonly/1,
striphead_to_details/1,
key_compare/3,
key_dominates/2,
print_key/1]).
-include_lib("eunit/include/eunit.hrl").
@ -172,17 +177,20 @@ book_start(Opts) ->
gen_server:start(?MODULE, [Opts], []).
book_riakput(Pid, Object, IndexSpecs) ->
PrimaryKey = {o, Object#r_object.bucket, Object#r_object.key},
PrimaryKey = {o, Object#r_object.bucket, Object#r_object.key, null},
gen_server:call(Pid, {put, PrimaryKey, Object, IndexSpecs}, infinity).
book_riakget(Pid, Bucket, Key) ->
PrimaryKey = {o, Bucket, Key},
PrimaryKey = {o, Bucket, Key, null},
gen_server:call(Pid, {get, PrimaryKey}, infinity).
book_riakhead(Pid, Bucket, Key) ->
PrimaryKey = {o, Bucket, Key},
PrimaryKey = {o, Bucket, Key, null},
gen_server:call(Pid, {head, PrimaryKey}, infinity).
book_returnfolder(Pid, FolderType) ->
gen_server:call(Pid, {return_folder, FolderType}, infinity).
book_snapshotstore(Pid, Requestor, Timeout) ->
gen_server:call(Pid, {snapshot, Requestor, store, Timeout}, infinity).
@ -307,6 +315,15 @@ handle_call({snapshot, _Requestor, SnapType, _Timeout}, _From, State) ->
null},
State}
end;
handle_call({return_folder, FolderType}, _From, State) ->
case FolderType of
{bucket_stats, Bucket} ->
{reply,
bucket_stats(State#state.penciller,
State#state.ledger_cache,
Bucket),
State}
end;
handle_call({compact_journal, Timeout}, _From, State) ->
ok = leveled_inker:ink_compactjournal(State#state.inker,
self(),
@ -343,6 +360,26 @@ code_change(_OldVsn, State, _Extra) ->
%%% Internal functions
%%%============================================================================
bucket_stats(Penciller, LedgerCache, Bucket) ->
Folder = fun() ->
PCLopts = #penciller_options{start_snapshot=true,
source_penciller=Penciller},
{ok, LedgerSnapshot} = leveled_penciller:pcl_start(PCLopts),
Increment = gb_trees:to_list(LedgerCache),
ok = leveled_penciller:pcl_loadsnapshot(LedgerSnapshot,
Increment),
StartKey = {o, Bucket, null, null},
EndKey = {o, Bucket, null, null},
Acc = leveled_penciller:pcl_fetchkeys(LedgerSnapshot,
StartKey,
EndKey,
fun accumulate_size/3,
{0, 0}),
ok = leveled_penciller:pcl_close(LedgerSnapshot),
Acc
end,
{async, Folder}.
shutdown_wait([], _Inker) ->
false;
shutdown_wait([TopPause|Rest], Inker) ->
@ -411,12 +448,30 @@ strip_to_keyonly({K, _V}) -> K.
strip_to_keyseqonly({K, {SeqN, _, _}}) -> {K, SeqN}.
strip_to_keyseqstatusonly({K, {SeqN, St, _MD}}) -> {K, SeqN, St}.
strip_to_statusonly({_, {_, St, _}}) -> St.
strip_to_seqonly({_, {SeqN, _, _}}) -> SeqN.
striphead_to_details({SeqN, St, MD}) -> {SeqN, St, MD}.
key_dominates(LeftKey, RightKey) ->
case {LeftKey, RightKey} of
{{LK, _LVAL}, {RK, _RVAL}} when LK < RK ->
left_hand_first;
{{LK, _LVAL}, {RK, _RVAL}} when RK < LK ->
right_hand_first;
{{LK, {LSN, _LST, _LMD}}, {RK, {RSN, _RST, _RMD}}}
when LK == RK, LSN >= RSN ->
left_hand_dominant;
{{LK, {LSN, _LST, _LMD}}, {RK, {RSN, _RST, _RMD}}}
when LK == RK, LSN < RSN ->
right_hand_dominant
end.
get_metadatas(#r_object{contents=Contents}) ->
[Content#r_content.metadata || Content <- Contents].
@ -435,8 +490,13 @@ hash(Obj=#r_object{}) ->
extract_metadata(Obj, Size) ->
{get_metadatas(Obj), vclock(Obj), hash(Obj), Size}.
accumulate_size(_Key, Value, {Size, Count}) ->
{_, _, MD} = Value,
{_, _, _, ObjSize} = MD,
{Size + ObjSize, Count + 1}.
build_metadata_object(PrimaryKey, Head) ->
{o, Bucket, Key} = PrimaryKey,
{o, Bucket, Key, null} = PrimaryKey,
{MD, VC, _, _} = Head,
Contents = lists:foldl(fun(X, Acc) -> Acc ++ [#r_content{metadata=X}] end,
[],
@ -453,12 +513,36 @@ convert_indexspecs(IndexSpecs, SQN, PrimaryKey) ->
%% TODO: timestamps for delayed reaping
{tomb, infinity}
end,
{o, B, K} = PrimaryKey,
{{i, B, IndexField, IndexValue, K},
{o, B, K, _SK} = PrimaryKey,
{{i, B, {IndexField, IndexValue}, K},
{SQN, Status, null}}
end,
IndexSpecs).
% Return a tuple of string to ease the printing of keys to logs
print_key(Key) ->
case Key of
{o, B, K, _SK} ->
{"Object", B, K};
{i, B, {F, _V}, _K} ->
{"Index", B, F}
end.
% Compare a key against a query key, only comparing elements that are non-null
% in the Query key
key_compare(QueryKey, CheckingKey, gt) ->
key_compare(QueryKey, CheckingKey, fun(X,Y) -> X > Y end);
key_compare(QueryKey, CheckingKey, lt) ->
key_compare(QueryKey, CheckingKey, fun(X,Y) -> X < Y end);
key_compare({QK1, null, null, null}, {CK1, _, _, _}, CompareFun) ->
CompareFun(QK1, CK1);
key_compare({QK1, QK2, null, null}, {CK1, CK2, _, _}, CompareFun) ->
CompareFun({QK1, QK2}, {CK1, CK2});
key_compare({QK1, QK2, QK3, null}, {CK1, CK2, CK3, _}, CompareFun) ->
CompareFun({QK1, QK2, QK3}, {CK1, CK2, CK3});
key_compare(QueryKey, CheckingKey, CompareFun) ->
CompareFun(QueryKey, CheckingKey).
preparefor_ledgercache(PK, SQN, Obj, Size, IndexSpecs) ->
PrimaryChange = {PK,
@ -628,12 +712,12 @@ indexspecs_test() ->
IndexSpecs = [{add, "t1_int", 456},
{add, "t1_bin", "adbc123"},
{remove, "t1_bin", "abdc456"}],
Changes = convert_indexspecs(IndexSpecs, 1, {o, "Bucket", "Key2"}),
?assertMatch({{i, "Bucket", "t1_int", 456, "Key2"},
Changes = convert_indexspecs(IndexSpecs, 1, {o, "Bucket", "Key2", null}),
?assertMatch({{i, "Bucket", {"t1_int", 456}, "Key2"},
{1, {active, infinity}, null}}, lists:nth(1, Changes)),
?assertMatch({{i, "Bucket", "t1_bin", "adbc123", "Key2"},
?assertMatch({{i, "Bucket", {"t1_bin", "adbc123"}, "Key2"},
{1, {active, infinity}, null}}, lists:nth(2, Changes)),
?assertMatch({{i, "Bucket", "t1_bin", "abdc456", "Key2"},
?assertMatch({{i, "Bucket", {"t1_bin", "abdc456"}, "Key2"},
{1, {tomb, infinity}, null}}, lists:nth(3, Changes)).
-endif.

View file

@ -251,8 +251,8 @@ handle_call({register_snapshot, Requestor}, _From , State) ->
State#state{registered_snapshots=Rs}};
handle_call({release_snapshot, Snapshot}, _From , State) ->
Rs = lists:keydelete(Snapshot, 1, State#state.registered_snapshots),
io:format("Snapshot ~w released~n", [Snapshot]),
io:format("Remaining snapshots are ~w~n", [Rs]),
io:format("Ledger snapshot ~w released~n", [Snapshot]),
io:format("Remaining ledger snapshots are ~w~n", [Rs]),
{reply, ok, State#state{registered_snapshots=Rs}};
handle_call(get_manifest, _From, State) ->
{reply, State#state.manifest, State};

View file

@ -100,6 +100,7 @@ init([]) ->
handle_call({register, Owner}, _From, State) ->
{reply, ok, State#state{owner=Owner}, ?INACTIVITY_TIMEOUT};
handle_call({manifest_change, return, true}, _From, State) ->
io:format("Request for manifest change from clerk on closing~n"),
case State#state.change_pending of
true ->
WI = State#state.work_item,
@ -110,11 +111,13 @@ handle_call({manifest_change, return, true}, _From, State) ->
handle_call({manifest_change, confirm, Closing}, From, State) ->
case Closing of
true ->
io:format("Confirmation of manifest change on closing~n"),
WI = State#state.work_item,
ok = mark_for_delete(WI#penciller_work.unreferenced_files,
State#state.owner),
{stop, normal, ok, State};
false ->
io:format("Prompted confirmation of manifest change~n"),
gen_server:reply(From, ok),
WI = State#state.work_item,
mark_for_delete(WI#penciller_work.unreferenced_files,
@ -168,6 +171,7 @@ requestandhandle_work(State) ->
{NewManifest, FilesToDelete} = merge(WI),
UpdWI = WI#penciller_work{new_manifest=NewManifest,
unreferenced_files=FilesToDelete},
io:format("Clerk prompting Penciller regarding manifest change~n"),
ok = leveled_penciller:pcl_promptmanifestchange(State#state.owner,
UpdWI),
{true, UpdWI}

View file

@ -234,12 +234,14 @@
pcl_start/1,
pcl_pushmem/2,
pcl_fetch/2,
pcl_fetchkeys/5,
pcl_checksequencenumber/3,
pcl_workforclerk/1,
pcl_promptmanifestchange/2,
pcl_confirmdelete/2,
pcl_close/1,
pcl_registersnapshot/2,
pcl_releasesnapshot/2,
pcl_updatesnapshotcache/3,
pcl_loadsnapshot/2,
pcl_getstartupsequencenumber/1,
@ -284,7 +286,6 @@
snapshot_fully_loaded = false :: boolean(),
source_penciller :: pid()}).
%%%============================================================================
%%% API
@ -301,6 +302,11 @@ pcl_pushmem(Pid, DumpList) ->
pcl_fetch(Pid, Key) ->
gen_server:call(Pid, {fetch, Key}, infinity).
pcl_fetchkeys(Pid, StartKey, EndKey, AccFun, InitAcc) ->
gen_server:call(Pid,
{fetch_keys, StartKey, EndKey, AccFun, InitAcc},
infinity).
pcl_checksequencenumber(Pid, Key, SQN) ->
gen_server:call(Pid, {check_sqn, Key, SQN}, infinity).
@ -319,6 +325,9 @@ pcl_getstartupsequencenumber(Pid) ->
pcl_registersnapshot(Pid, Snapshot) ->
gen_server:call(Pid, {register_snapshot, Snapshot}, infinity).
pcl_releasesnapshot(Pid, Snapshot) ->
gen_server:cast(Pid, {release_snapshot, Snapshot}).
pcl_updatesnapshotcache(Pid, Tree, SQN) ->
gen_server:cast(Pid, {update_snapshotcache, Tree, SQN}).
@ -476,6 +485,16 @@ handle_call({check_sqn, Key, SQN},
State#state.levelzero_snapshot),
SQN),
State};
handle_call({fetch_keys, StartKey, EndKey, AccFun, InitAcc},
_From,
State=#state{snapshot_fully_loaded=Ready})
when Ready == true ->
L0iter = gb_trees:iterator_from(StartKey, State#state.levelzero_snapshot),
SFTiter = initiate_rangequery_frommanifest(StartKey,
EndKey,
State#state.manifest),
Acc = keyfolder(L0iter, SFTiter, StartKey, EndKey, {AccFun, InitAcc}),
{reply, Acc, State};
handle_call(work_for_clerk, From, State) ->
{UpdState, Work} = return_work(State, From),
{reply, {Work, UpdState#state.backlog}, UpdState};
@ -498,7 +517,10 @@ handle_call({load_snapshot, Increment}, _From, State) ->
TreeSQN0 > MemTableCopy#l0snapshot.ledger_sqn ->
pcl_updatesnapshotcache(State#state.source_penciller,
Tree0,
TreeSQN0)
TreeSQN0);
true ->
io:format("No update required to snapshot cache~n"),
ok
end,
{Tree1, TreeSQN1} = roll_new_tree(Tree0, [Increment], TreeSQN0),
io:format("Snapshot loaded to start at SQN~w~n", [TreeSQN1]),
@ -517,15 +539,20 @@ handle_cast({manifest_change, WI}, State) ->
confirm,
false),
{noreply, UpdState};
handle_cast(_Msg, State) ->
{noreply, State}.
handle_cast({release_snapshot, Snapshot}, State) ->
Rs = lists:keydelete(Snapshot, 1, State#state.registered_snapshots),
io:format("Penciller snapshot ~w released~n", [Snapshot]),
{noreply, State#state{registered_snapshots=Rs}}.
handle_info(_Info, State) ->
{noreply, State}.
terminate(_Reason, _State=#state{is_snapshot=Snap}) when Snap == true ->
terminate(Reason, State=#state{is_snapshot=Snap}) when Snap == true ->
ok = pcl_releasesnapshot(State#state.source_penciller, self()),
io:format("Sent release message for snapshot following close for "
++ "reason ~w~n", [Reason]),
ok;
terminate(_Reason, State) ->
terminate(Reason, State) ->
%% When a Penciller shuts down it isn't safe to try an manage the safe
%% finishing of any outstanding work. The last commmitted manifest will
%% be used.
@ -542,6 +569,7 @@ terminate(_Reason, State) ->
%% The cast may not succeed as the clerk could be synchronously calling
%% the penciller looking for a manifest commit
%%
io:format("Penciller closing for reason - ~w~n", [Reason]),
MC = leveled_pclerk:clerk_manifestchange(State#state.clerk,
return,
true),
@ -976,19 +1004,216 @@ print_manifest(Manifest) ->
io:format("Manifest at Level ~w~n", [L]),
Level = get_item(L, Manifest, []),
lists:foreach(fun(M) ->
{_, SB, SK} = M#manifest_entry.start_key,
{_, EB, EK} = M#manifest_entry.end_key,
io:format("Manifest entry of " ++
"startkey ~s ~s " ++
"endkey ~s ~s " ++
"filename=~s~n",
[SB, SK, EB, EK,
M#manifest_entry.filename])
end,
print_manifest_entry(M) end,
Level)
end,
lists:seq(1, ?MAX_LEVELS - 1)).
print_manifest_entry(Entry) ->
{S1, S2, S3} = leveled_bookie:print_key(Entry#manifest_entry.start_key),
{E1, E2, E3} = leveled_bookie:print_key(Entry#manifest_entry.end_key),
io:format("Manifest entry of " ++
"startkey ~s ~s ~s " ++
"endkey ~s ~s ~s " ++
"filename=~s~n",
[S1, S2, S3, E1, E2, E3,
Entry#manifest_entry.filename]).
initiate_rangequery_frommanifest(StartKey, EndKey, Manifest) ->
CompareFun = fun(M) ->
C1 = leveled_bookie:key_compare(StartKey,
M#manifest_entry.end_key,
gt),
C2 = leveled_bookie:key_compare(EndKey,
M#manifest_entry.start_key,
lt),
not (C1 or C2) end,
lists:foldl(fun(L, AccL) ->
Level = get_item(L, Manifest, []),
FL = lists:foldl(fun(M, Acc) ->
case CompareFun(M) of
true ->
Acc ++ [{next_file, M}];
false ->
Acc
end end,
[],
Level),
case FL of
[] -> AccL;
FL -> AccL ++ [{L, FL}]
end
end,
[],
lists:seq(1, ?MAX_LEVELS - 1)).
%% Looks to find the best choice for the next key across the levels (other
%% than in-memory table)
%% In finding the best choice, the next key in a given level may be a next
%% block or next file pointer which will need to be expanded
find_nextkey(QueryArray, StartKey, EndKey) ->
find_nextkey(QueryArray,
1,
{null, null},
{fun leveled_sft:sft_getkvrange/4, StartKey, EndKey, 1}).
find_nextkey(_QueryArray, LCnt, {null, null}, _QueryFunT)
when LCnt > ?MAX_LEVELS ->
% The array has been scanned wihtout finding a best key - must be
% exhausted - respond to indicate no more keys to be found by the
% iterator
no_more_keys;
find_nextkey(QueryArray, LCnt, {BKL, BestKV}, _QueryFunT)
when LCnt > ?MAX_LEVELS ->
% All levels have been scanned, so need to remove the best result from
% the array, and return that array along with the best key/sqn/status
% combination
{BKL, [BestKV|Tail]} = lists:keyfind(BKL, 1, QueryArray),
{lists:keyreplace(BKL, 1, QueryArray, {BKL, Tail}), BestKV};
find_nextkey(QueryArray, LCnt, {BestKeyLevel, BestKV}, QueryFunT) ->
% Get the next key at this level
{NextKey, RestOfKeys} = case lists:keyfind(LCnt, 1, QueryArray) of
false ->
{null, null};
{LCnt, []} ->
{null, null};
{LCnt, [NK|ROfKs]} ->
{NK, ROfKs}
end,
% Compare the next key at this level with the best key
case {NextKey, BestKeyLevel, BestKV} of
{null, BKL, BKV} ->
% There is no key at this level - go to the next level
find_nextkey(QueryArray, LCnt + 1, {BKL, BKV}, QueryFunT);
{{next_file, ManifestEntry}, BKL, BKV} ->
% The first key at this level is pointer to a file - need to query
% the file to expand this level out before proceeding
Owner = ManifestEntry#manifest_entry.owner,
{QueryFun, StartKey, EndKey, ScanSize} = QueryFunT,
QueryResult = QueryFun(Owner, StartKey, EndKey, ScanSize),
NewEntry = {LCnt, QueryResult ++ RestOfKeys},
% Need to loop around at this level (LCnt) as we have not yet
% examined a real key at this level
find_nextkey(lists:keyreplace(LCnt, 1, QueryArray, NewEntry),
LCnt,
{BKL, BKV},
QueryFunT);
{{next, SFTpid, NewStartKey}, BKL, BKV} ->
% The first key at this level is pointer within a file - need to
% query the file to expand this level out before proceeding
{QueryFun, _StartKey, EndKey, ScanSize} = QueryFunT,
QueryResult = QueryFun(SFTpid, NewStartKey, EndKey, ScanSize),
NewEntry = {LCnt, QueryResult ++ RestOfKeys},
% Need to loop around at this level (LCnt) as we have not yet
% examined a real key at this level
find_nextkey(lists:keyreplace(LCnt, 1, QueryArray, NewEntry),
LCnt,
{BKL, BKV},
QueryFunT);
{{Key, Val}, null, null} ->
% No best key set - so can assume that this key is the best key,
% and check the higher levels
find_nextkey(QueryArray,
LCnt + 1,
{LCnt, {Key, Val}},
QueryFunT);
{{Key, Val}, _BKL, {BestKey, _BestVal}} when Key < BestKey ->
% There is a real key and a best key to compare, and the real key
% at this level is before the best key, and so is now the new best
% key
% The QueryArray is not modified until we have checked all levels
find_nextkey(QueryArray,
LCnt + 1,
{LCnt, {Key, Val}},
QueryFunT);
{{Key, Val}, BKL, {BestKey, BestVal}} when Key == BestKey ->
SQN = leveled_bookie:strip_to_seqonly({Key, Val}),
BestSQN = leveled_bookie:strip_to_seqonly({BestKey, BestVal}),
if
SQN =< BestSQN ->
% This is a dominated key, so we need to skip over it
NewEntry = {LCnt, RestOfKeys},
find_nextkey(lists:keyreplace(LCnt, 1, QueryArray, NewEntry),
LCnt + 1,
{BKL, {BestKey, BestVal}},
QueryFunT);
SQN > BestSQN ->
% There is a real key at the front of this level and it has
% a higher SQN than the best key, so we should use this as
% the best key
% But we also need to remove the dominated key from the
% lower level in the query array
io:format("Key at level ~w with SQN ~w is better than " ++
"key at lower level ~w with SQN ~w~n",
[LCnt, SQN, BKL, BestSQN]),
OldBestEntry = lists:keyfind(BKL, 1, QueryArray),
{BKL, [{BestKey, BestVal}|BestTail]} = OldBestEntry,
find_nextkey(lists:keyreplace(BKL,
1,
QueryArray,
{BKL, BestTail}),
LCnt + 1,
{LCnt, {Key, Val}},
QueryFunT)
end;
{_, BKL, BKV} ->
% This is not the best key
find_nextkey(QueryArray, LCnt + 1, {BKL, BKV}, QueryFunT)
end.
keyfolder(null, SFTiterator, StartKey, EndKey, {AccFun, Acc}) ->
case find_nextkey(SFTiterator, StartKey, EndKey) of
no_more_keys ->
Acc;
{NxtSFTiterator, {SFTKey, SFTVal}} ->
Acc1 = AccFun(SFTKey, SFTVal, Acc),
keyfolder(null, NxtSFTiterator, StartKey, EndKey, {AccFun, Acc1})
end;
keyfolder(IMMiterator, SFTiterator, StartKey, EndKey, {AccFun, Acc}) ->
case gb_trees:next(IMMiterator) of
none ->
% There are no more keys in the in-memory iterator, so now
% iterate only over the remaining keys in the SFT iterator
keyfolder(null, SFTiterator, StartKey, EndKey, {AccFun, Acc});
{IMMKey, IMMVal, NxtIMMiterator} ->
case {leveled_bookie:key_compare(EndKey, IMMKey, lt),
find_nextkey(SFTiterator, StartKey, EndKey)} of
{true, _} ->
% There are no more keys in-range in the in-memory
% iterator, so take action as if this iterator is empty
% (see above)
keyfolder(null, SFTiterator,
StartKey, EndKey, {AccFun, Acc});
{false, no_more_keys} ->
% No more keys in range in the persisted store, so use the
% in-memory KV as the next
Acc1 = AccFun(IMMKey, IMMVal, Acc),
keyfolder(NxtIMMiterator, SFTiterator,
StartKey, EndKey, {AccFun, Acc1});
{false, {NxtSFTiterator, {SFTKey, SFTVal}}} ->
% There is a next key, so need to know which is the next
% key between the two (and handle two keys with different
% sequence numbers).
case leveled_bookie:key_dominates({IMMKey, IMMVal},
{SFTKey, SFTVal}) of
left_hand_first ->
Acc1 = AccFun(IMMKey, IMMVal, Acc),
keyfolder(NxtIMMiterator, SFTiterator,
StartKey, EndKey, {AccFun, Acc1});
right_hand_first ->
Acc1 = AccFun(SFTKey, SFTVal, Acc),
keyfolder(IMMiterator, NxtSFTiterator,
StartKey, EndKey, {AccFun, Acc1});
left_hand_dominant ->
Acc1 = AccFun(IMMKey, IMMVal, Acc),
keyfolder(NxtIMMiterator, NxtSFTiterator,
StartKey, EndKey, {AccFun, Acc1})
end
end
end.
assess_workqueue(WorkQ, ?MAX_LEVELS - 1, _Manifest) ->
WorkQ;
@ -1069,8 +1294,14 @@ commit_manifest_change(ReturnedWorkItem, State) ->
rename_manifest_files(RootPath, NewMSN) ->
file:rename(filepath(RootPath, NewMSN, pending_manifest),
filepath(RootPath, NewMSN, current_manifest)).
OldFN = filepath(RootPath, NewMSN, pending_manifest),
NewFN = filepath(RootPath, NewMSN, current_manifest),
io:format("Rename of manifest from ~s ~w to ~s ~w~n",
[OldFN,
filelib:is_file(OldFN),
NewFN,
filelib:is_file(NewFN)]),
file:rename(OldFN,NewFN).
filepath(RootPath, manifest) ->
RootPath ++ "/" ++ ?MANIFEST_FP;
@ -1152,20 +1383,27 @@ clean_subdir(DirPath) ->
end.
compaction_work_assessment_test() ->
L0 = [{{o, "B1", "K1"}, {o, "B3", "K3"}, dummy_pid}],
L1 = [{{o, "B1", "K1"}, {o, "B2", "K2"}, dummy_pid},
{{o, "B2", "K3"}, {o, "B4", "K4"}, dummy_pid}],
L0 = [{{o, "B1", "K1", null}, {o, "B3", "K3", null}, dummy_pid}],
L1 = [{{o, "B1", "K1", null}, {o, "B2", "K2", null}, dummy_pid},
{{o, "B2", "K3", null}, {o, "B4", "K4", null}, dummy_pid}],
Manifest = [{0, L0}, {1, L1}],
WorkQ1 = assess_workqueue([], 0, Manifest),
?assertMatch(WorkQ1, [{0, Manifest}]),
L1Alt = lists:append(L1,
[{{o, "B5", "K0001"}, {o, "B5", "K9999"}, dummy_pid},
{{o, "B6", "K0001"}, {o, "B6", "K9999"}, dummy_pid},
{{o, "B7", "K0001"}, {o, "B7", "K9999"}, dummy_pid},
{{o, "B8", "K0001"}, {o, "B8", "K9999"}, dummy_pid},
{{o, "B9", "K0001"}, {o, "B9", "K9999"}, dummy_pid},
{{o, "BA", "K0001"}, {o, "BA", "K9999"}, dummy_pid},
{{o, "BB", "K0001"}, {o, "BB", "K9999"}, dummy_pid}]),
[{{o, "B5", "K0001", null}, {o, "B5", "K9999", null},
dummy_pid},
{{o, "B6", "K0001", null}, {o, "B6", "K9999", null},
dummy_pid},
{{o, "B7", "K0001", null}, {o, "B7", "K9999", null},
dummy_pid},
{{o, "B8", "K0001", null}, {o, "B8", "K9999", null},
dummy_pid},
{{o, "B9", "K0001", null}, {o, "B9", "K9999", null},
dummy_pid},
{{o, "BA", "K0001", null}, {o, "BA", "K9999", null},
dummy_pid},
{{o, "BB", "K0001", null}, {o, "BB", "K9999", null},
dummy_pid}]),
Manifest3 = [{0, []}, {1, L1Alt}],
WorkQ3 = assess_workqueue([], 0, Manifest3),
?assertMatch(WorkQ3, [{1, Manifest3}]).
@ -1199,26 +1437,26 @@ simple_server_test() ->
clean_testdir(RootPath),
{ok, PCL} = pcl_start(#penciller_options{root_path=RootPath,
max_inmemory_tablesize=1000}),
Key1 = {{o,"Bucket0001", "Key0001"}, {1, {active, infinity}, null}},
Key1 = {{o,"Bucket0001", "Key0001", null}, {1, {active, infinity}, null}},
KL1 = lists:sort(leveled_sft:generate_randomkeys({1000, 2})),
Key2 = {{o,"Bucket0002", "Key0002"}, {1002, {active, infinity}, null}},
Key2 = {{o,"Bucket0002", "Key0002", null}, {1002, {active, infinity}, null}},
KL2 = lists:sort(leveled_sft:generate_randomkeys({1000, 1002})),
Key3 = {{o,"Bucket0003", "Key0003"}, {2002, {active, infinity}, null}},
Key3 = {{o,"Bucket0003", "Key0003", null}, {2002, {active, infinity}, null}},
KL3 = lists:sort(leveled_sft:generate_randomkeys({1000, 2002})),
Key4 = {{o,"Bucket0004", "Key0004"}, {3002, {active, infinity}, null}},
Key4 = {{o,"Bucket0004", "Key0004", null}, {3002, {active, infinity}, null}},
KL4 = lists:sort(leveled_sft:generate_randomkeys({1000, 3002})),
ok = pcl_pushmem(PCL, [Key1]),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001"})),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001", null})),
ok = pcl_pushmem(PCL, KL1),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001"})),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001", null})),
maybe_pause_push(pcl_pushmem(PCL, [Key2])),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001"})),
?assertMatch(Key2, pcl_fetch(PCL, {o,"Bucket0002", "Key0002"})),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001", null})),
?assertMatch(Key2, pcl_fetch(PCL, {o,"Bucket0002", "Key0002", null})),
maybe_pause_push(pcl_pushmem(PCL, KL2)),
maybe_pause_push(pcl_pushmem(PCL, [Key3])),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001"})),
?assertMatch(Key2, pcl_fetch(PCL, {o,"Bucket0002", "Key0002"})),
?assertMatch(Key3, pcl_fetch(PCL, {o,"Bucket0003", "Key0003"})),
?assertMatch(Key1, pcl_fetch(PCL, {o,"Bucket0001", "Key0001", null})),
?assertMatch(Key2, pcl_fetch(PCL, {o,"Bucket0002", "Key0002", null})),
?assertMatch(Key3, pcl_fetch(PCL, {o,"Bucket0003", "Key0003", null})),
ok = pcl_close(PCL),
{ok, PCLr} = pcl_start(#penciller_options{root_path=RootPath,
max_inmemory_tablesize=1000}),
@ -1233,61 +1471,86 @@ simple_server_test() ->
%% everything got persisted
ok;
_ ->
io:format("Unexpected sequence number on restart ~w~n", [TopSQN]),
io:format("Unexpected sequence number on restart ~w~n",
[TopSQN]),
error
end,
?assertMatch(ok, Check),
?assertMatch(Key1, pcl_fetch(PCLr, {o,"Bucket0001", "Key0001"})),
?assertMatch(Key2, pcl_fetch(PCLr, {o,"Bucket0002", "Key0002"})),
?assertMatch(Key3, pcl_fetch(PCLr, {o,"Bucket0003", "Key0003"})),
?assertMatch(Key1, pcl_fetch(PCLr, {o,"Bucket0001", "Key0001", null})),
?assertMatch(Key2, pcl_fetch(PCLr, {o,"Bucket0002", "Key0002", null})),
?assertMatch(Key3, pcl_fetch(PCLr, {o,"Bucket0003", "Key0003", null})),
maybe_pause_push(pcl_pushmem(PCLr, KL3)),
maybe_pause_push(pcl_pushmem(PCLr, [Key4])),
maybe_pause_push(pcl_pushmem(PCLr, KL4)),
?assertMatch(Key1, pcl_fetch(PCLr, {o,"Bucket0001", "Key0001"})),
?assertMatch(Key2, pcl_fetch(PCLr, {o,"Bucket0002", "Key0002"})),
?assertMatch(Key3, pcl_fetch(PCLr, {o,"Bucket0003", "Key0003"})),
?assertMatch(Key4, pcl_fetch(PCLr, {o,"Bucket0004", "Key0004"})),
?assertMatch(Key1, pcl_fetch(PCLr, {o,"Bucket0001", "Key0001", null})),
?assertMatch(Key2, pcl_fetch(PCLr, {o,"Bucket0002", "Key0002", null})),
?assertMatch(Key3, pcl_fetch(PCLr, {o,"Bucket0003", "Key0003", null})),
?assertMatch(Key4, pcl_fetch(PCLr, {o,"Bucket0004", "Key0004", null})),
SnapOpts = #penciller_options{start_snapshot = true,
source_penciller = PCLr},
{ok, PclSnap} = pcl_start(SnapOpts),
ok = pcl_loadsnapshot(PclSnap, []),
?assertMatch(Key1, pcl_fetch(PclSnap, {o,"Bucket0001", "Key0001"})),
?assertMatch(Key2, pcl_fetch(PclSnap, {o,"Bucket0002", "Key0002"})),
?assertMatch(Key3, pcl_fetch(PclSnap, {o,"Bucket0003", "Key0003"})),
?assertMatch(Key4, pcl_fetch(PclSnap, {o,"Bucket0004", "Key0004"})),
?assertMatch(Key1, pcl_fetch(PclSnap, {o,"Bucket0001", "Key0001", null})),
?assertMatch(Key2, pcl_fetch(PclSnap, {o,"Bucket0002", "Key0002", null})),
?assertMatch(Key3, pcl_fetch(PclSnap, {o,"Bucket0003", "Key0003", null})),
?assertMatch(Key4, pcl_fetch(PclSnap, {o,"Bucket0004", "Key0004", null})),
?assertMatch(true, pcl_checksequencenumber(PclSnap,
{o,"Bucket0001", "Key0001"},
{o,
"Bucket0001",
"Key0001",
null},
1)),
?assertMatch(true, pcl_checksequencenumber(PclSnap,
{o,"Bucket0002", "Key0002"},
{o,
"Bucket0002",
"Key0002",
null},
1002)),
?assertMatch(true, pcl_checksequencenumber(PclSnap,
{o,"Bucket0003", "Key0003"},
{o,
"Bucket0003",
"Key0003",
null},
2002)),
?assertMatch(true, pcl_checksequencenumber(PclSnap,
{o,"Bucket0004", "Key0004"},
{o,
"Bucket0004",
"Key0004",
null},
3002)),
% Add some more keys and confirm that chekc sequence number still
% sees the old version in the previous snapshot, but will see the new version
% in a new snapshot
Key1A = {{o,"Bucket0001", "Key0001"}, {4002, {active, infinity}, null}},
Key1A = {{o,"Bucket0001", "Key0001", null}, {4002, {active, infinity}, null}},
KL1A = lists:sort(leveled_sft:generate_randomkeys({4002, 2})),
maybe_pause_push(pcl_pushmem(PCLr, [Key1A])),
maybe_pause_push(pcl_pushmem(PCLr, KL1A)),
?assertMatch(true, pcl_checksequencenumber(PclSnap,
{o,"Bucket0001", "Key0001"},
{o,
"Bucket0001",
"Key0001",
null},
1)),
ok = pcl_close(PclSnap),
{ok, PclSnap2} = pcl_start(SnapOpts),
ok = pcl_loadsnapshot(PclSnap2, []),
?assertMatch(false, pcl_checksequencenumber(PclSnap2,
{o,"Bucket0001", "Key0001"},
{o,
"Bucket0001",
"Key0001",
null},
1)),
?assertMatch(true, pcl_checksequencenumber(PclSnap2,
{o,"Bucket0001", "Key0001"},
{o,
"Bucket0001",
"Key0001",
null},
4002)),
?assertMatch(true, pcl_checksequencenumber(PclSnap2,
{o,"Bucket0002", "Key0002"},
{o,
"Bucket0002",
"Key0002",
null},
1002)),
ok = pcl_close(PclSnap2),
ok = pcl_close(PCLr),
@ -1334,4 +1597,174 @@ memcopy_updatecache_test() ->
?assertMatch(2000, Size2),
?assertMatch(3000, MemCopy4#l0snapshot.ledger_sqn).
rangequery_manifest_test() ->
{E1,
E2,
E3} = {#manifest_entry{start_key={i, "Bucket1", {"Idx1", "Fld1"}, "K8"},
end_key={i, "Bucket1", {"Idx1", "Fld9"}, "K93"},
filename="Z1"},
#manifest_entry{start_key={i, "Bucket1", {"Idx1", "Fld9"}, "K97"},
end_key={o, "Bucket1", "K71", null},
filename="Z2"},
#manifest_entry{start_key={o, "Bucket1", "K75", null},
end_key={o, "Bucket1", "K993", null},
filename="Z3"}},
{E4,
E5,
E6} = {#manifest_entry{start_key={i, "Bucket1", {"Idx1", "Fld1"}, "K8"},
end_key={i, "Bucket1", {"Idx1", "Fld7"}, "K93"},
filename="Z4"},
#manifest_entry{start_key={i, "Bucket1", {"Idx1", "Fld7"}, "K97"},
end_key={o, "Bucket1", "K78", null},
filename="Z5"},
#manifest_entry{start_key={o, "Bucket1", "K81", null},
end_key={o, "Bucket1", "K996", null},
filename="Z6"}},
Man = [{1, [E1, E2, E3]}, {2, [E4, E5, E6]}],
R1 = initiate_rangequery_frommanifest({o, "Bucket1", "K711", null},
{o, "Bucket1", "K999", null},
Man),
?assertMatch([{1, [{next_file, E3}]},
{2, [{next_file, E5}, {next_file, E6}]}],
R1),
R2 = initiate_rangequery_frommanifest({i, "Bucket1", {"Idx1", "Fld8"}, null},
{i, "Bucket1", {"Idx1", "Fld8"}, null},
Man),
?assertMatch([{1, [{next_file, E1}]}, {2, [{next_file, E5}]}],
R2),
R3 = initiate_rangequery_frommanifest({i, "Bucket1", {"Idx0", "Fld8"}, null},
{i, "Bucket1", {"Idx0", "Fld9"}, null},
Man),
?assertMatch([], R3).
simple_findnextkey_test() ->
QueryArray = [
{2, [{{o, "Bucket1", "Key1"}, {5, {active, infinity}, null}},
{{o, "Bucket1", "Key5"}, {4, {active, infinity}, null}}]},
{3, [{{o, "Bucket1", "Key3"}, {3, {active, infinity}, null}}]},
{5, [{{o, "Bucket1", "Key2"}, {2, {active, infinity}, null}}]}
],
{Array2, KV1} = find_nextkey(QueryArray,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key1"}, {5, {active, infinity}, null}}, KV1),
{Array3, KV2} = find_nextkey(Array2,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key2"}, {2, {active, infinity}, null}}, KV2),
{Array4, KV3} = find_nextkey(Array3,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key3"}, {3, {active, infinity}, null}}, KV3),
{Array5, KV4} = find_nextkey(Array4,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key5"}, {4, {active, infinity}, null}}, KV4),
ER = find_nextkey(Array5,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch(no_more_keys, ER).
sqnoverlap_findnextkey_test() ->
QueryArray = [
{2, [{{o, "Bucket1", "Key1"}, {5, {active, infinity}, null}},
{{o, "Bucket1", "Key5"}, {4, {active, infinity}, null}}]},
{3, [{{o, "Bucket1", "Key3"}, {3, {active, infinity}, null}}]},
{5, [{{o, "Bucket1", "Key5"}, {2, {active, infinity}, null}}]}
],
{Array2, KV1} = find_nextkey(QueryArray,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key1"}, {5, {active, infinity}, null}}, KV1),
{Array3, KV2} = find_nextkey(Array2,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key3"}, {3, {active, infinity}, null}}, KV2),
{Array4, KV3} = find_nextkey(Array3,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key5"}, {4, {active, infinity}, null}}, KV3),
ER = find_nextkey(Array4,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch(no_more_keys, ER).
sqnoverlap_otherway_findnextkey_test() ->
QueryArray = [
{2, [{{o, "Bucket1", "Key1"}, {5, {active, infinity}, null}},
{{o, "Bucket1", "Key5"}, {1, {active, infinity}, null}}]},
{3, [{{o, "Bucket1", "Key3"}, {3, {active, infinity}, null}}]},
{5, [{{o, "Bucket1", "Key5"}, {2, {active, infinity}, null}}]}
],
{Array2, KV1} = find_nextkey(QueryArray,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key1"}, {5, {active, infinity}, null}}, KV1),
{Array3, KV2} = find_nextkey(Array2,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key3"}, {3, {active, infinity}, null}}, KV2),
{Array4, KV3} = find_nextkey(Array3,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch({{o, "Bucket1", "Key5"}, {2, {active, infinity}, null}}, KV3),
ER = find_nextkey(Array4,
{o, "Bucket1", "Key0"},
{o, "Bucket1", "Key5"}),
?assertMatch(no_more_keys, ER).
foldwithimm_simple_test() ->
QueryArray = [
{2, [{{o, "Bucket1", "Key1"}, {5, {active, infinity}, null}},
{{o, "Bucket1", "Key5"}, {1, {active, infinity}, null}}]},
{3, [{{o, "Bucket1", "Key3"}, {3, {active, infinity}, null}}]},
{5, [{{o, "Bucket1", "Key5"}, {2, {active, infinity}, null}}]}
],
IMM0 = gb_trees:enter({o, "Bucket1", "Key6"},
{7, {active, infinity}, null},
gb_trees:empty()),
IMM1 = gb_trees:enter({o, "Bucket1", "Key1"},
{8, {active, infinity}, null},
IMM0),
IMM2 = gb_trees:enter({o, "Bucket1", "Key8"},
{9, {active, infinity}, null},
IMM1),
IMMiter = gb_trees:iterator_from({o, "Bucket1", "Key1"}, IMM2),
AccFun = fun(K, V, Acc) -> SQN= leveled_bookie:strip_to_seqonly({K, V}),
Acc ++ [{K, SQN}] end,
Acc = keyfolder(IMMiter,
QueryArray,
{o, "Bucket1", "Key1"}, {o, "Bucket1", "Key6"},
{AccFun, []}),
?assertMatch([{{o, "Bucket1", "Key1"}, 8},
{{o, "Bucket1", "Key3"}, 3},
{{o, "Bucket1", "Key5"}, 2},
{{o, "Bucket1", "Key6"}, 7}], Acc),
IMM1A = gb_trees:enter({o, "Bucket1", "Key1"},
{8, {active, infinity}, null},
gb_trees:empty()),
IMMiterA = gb_trees:iterator_from({o, "Bucket1", "Key1"}, IMM1A),
AccA = keyfolder(IMMiterA,
QueryArray,
{o, "Bucket1", "Key1"}, {o, "Bucket1", "Key6"},
{AccFun, []}),
?assertMatch([{{o, "Bucket1", "Key1"}, 8},
{{o, "Bucket1", "Key3"}, 3},
{{o, "Bucket1", "Key5"}, 2}], AccA),
IMM3 = gb_trees:enter({o, "Bucket1", "Key4"},
{10, {active, infinity}, null},
IMM2),
IMMiterB = gb_trees:iterator_from({o, "Bucket1", "Key1"}, IMM3),
AccB = keyfolder(IMMiterB,
QueryArray,
{o, "Bucket1", "Key1"}, {o, "Bucket1", "Key6"},
{AccFun, []}),
?assertMatch([{{o, "Bucket1", "Key1"}, 8},
{{o, "Bucket1", "Key3"}, 3},
{{o, "Bucket1", "Key4"}, 10},
{{o, "Bucket1", "Key5"}, 2},
{{o, "Bucket1", "Key6"}, 7}], AccB).
-endif.

View file

@ -14,8 +14,8 @@
%%
%% All keys are not equal in sft files, keys are only expected in a specific
%% series of formats
%% - {o, Bucket, Key} - Object Keys
%% - {i, Bucket, IndexName, IndexTerm, Key} - Postings
%% - {o, Bucket, Key, SubKey|null} - Object Keys
%% - {i, Bucket, {IndexName, IndexTerm}, Key} - Postings
%% The {Bucket, Key} part of all types of keys are hashed for segment filters.
%% For Postings the {Bucket, IndexName, IndexTerm} is also hashed. This
%% causes a false positive on lookup of a segment, but allows for the presence
@ -155,7 +155,7 @@
sft_new/5,
sft_open/1,
sft_get/2,
sft_getkeyrange/4,
sft_getkvrange/4,
sft_close/1,
sft_clear/1,
sft_checkready/1,
@ -243,15 +243,8 @@ sft_setfordelete(Pid, Penciller) ->
sft_get(Pid, Key) ->
gen_server:call(Pid, {get_kv, Key}, infinity).
sft_getkeyrange(Pid, StartKey, EndKey, ScanWidth) ->
gen_server:call(Pid,
{get_keyrange, StartKey, EndKey, ScanWidth},
infinity).
sft_getkvrange(Pid, StartKey, EndKey, ScanWidth) ->
gen_server:call(Pid,
{get_kvrange, StartKey, EndKey, ScanWidth},
infinity).
gen_server:call(Pid, {get_kvrange, StartKey, EndKey, ScanWidth}, infinity).
sft_clear(Pid) ->
gen_server:call(Pid, clear, infinity).
@ -313,15 +306,13 @@ handle_call({sft_open, Filename}, _From, _State) ->
handle_call({get_kv, Key}, _From, State) ->
Reply = fetch_keyvalue(State#state.handle, State, Key),
{reply, Reply, State};
handle_call({get_keyrange, StartKey, EndKey, ScanWidth}, _From, State) ->
Reply = fetch_range_keysonly(State#state.handle, State,
StartKey, EndKey,
ScanWidth),
{reply, Reply, State};
handle_call({get_kvrange, StartKey, EndKey, ScanWidth}, _From, State) ->
Reply = fetch_range_kv(State#state.handle, State,
StartKey, EndKey,
ScanWidth),
Reply = pointer_append_queryresults(fetch_range_kv(State#state.handle,
State,
StartKey,
EndKey,
ScanWidth),
self()),
{reply, Reply, State};
handle_call(close, _From, State) ->
{stop, normal, ok, State};
@ -582,7 +573,7 @@ acc_list_keysonly(null, empty) ->
acc_list_keysonly(null, RList) ->
RList;
acc_list_keysonly(R, RList) ->
lists:append(RList, [leveled_bookie:strip_to_keyseqonly(R)]).
lists:append(RList, [leveled_bookie:strip_to_keyseqstatusonly(R)]).
acc_list_kv(null, empty) ->
[];
@ -672,10 +663,12 @@ scan_block([], StartKey, _EndKey, _FunList, _AccFun, Acc) ->
{partial, Acc, StartKey};
scan_block([HeadKV|T], StartKey, EndKey, FunList, AccFun, Acc) ->
K = leveled_bookie:strip_to_keyonly(HeadKV),
case K of
K when K < StartKey, StartKey /= all ->
Pre = leveled_bookie:key_compare(StartKey, K, gt),
Post = leveled_bookie:key_compare(EndKey, K, lt),
case {Pre, Post} of
{true, _} when StartKey /= all ->
scan_block(T, StartKey, EndKey, FunList, AccFun, Acc);
K when K > EndKey, EndKey /= all ->
{_, true} when EndKey /= all ->
{complete, Acc};
_ ->
case applyfuns(FunList, HeadKV) of
@ -1121,8 +1114,7 @@ maybe_expand_pointer([H|Tail]) ->
case H of
{next, SFTPid, StartKey} ->
%% io:format("Scanning further on PID ~w ~w~n", [SFTPid, StartKey]),
QResult = sft_getkvrange(SFTPid, StartKey, all, ?MERGE_SCANWIDTH),
Acc = pointer_append_queryresults(QResult, SFTPid),
Acc = sft_getkvrange(SFTPid, StartKey, all, ?MERGE_SCANWIDTH),
lists:append(Acc, Tail);
_ ->
[H|Tail]
@ -1409,8 +1401,9 @@ generate_randomkeys(0, _SQN, Acc) ->
Acc;
generate_randomkeys(Count, SQN, Acc) ->
RandKey = {{o,
lists:concat(["Bucket", random:uniform(1024)]),
lists:concat(["Key", random:uniform(1024)])},
lists:concat(["Bucket", random:uniform(1024)]),
lists:concat(["Key", random:uniform(1024)]),
null},
{SQN,
{active, infinity}, null}},
generate_randomkeys(Count - 1, SQN + 1, [RandKey|Acc]).
@ -1423,73 +1416,74 @@ generate_sequentialkeys(Target, Incr, Acc) when Incr =:= Target ->
generate_sequentialkeys(Target, Incr, Acc) ->
KeyStr = string:right(integer_to_list(Incr), 8, $0),
NextKey = {{o,
"BucketSeq",
lists:concat(["Key", KeyStr])},
"BucketSeq",
lists:concat(["Key", KeyStr]),
null},
{5,
{active, infinity}, null}},
generate_sequentialkeys(Target, Incr + 1, [NextKey|Acc]).
simple_create_block_test() ->
KeyList1 = [{{o, "Bucket1", "Key1"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key3"}, {2, {active, infinity}, null}}],
KeyList2 = [{{o, "Bucket1", "Key2"}, {3, {active, infinity}, null}}],
KeyList1 = [{{o, "Bucket1", "Key1", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key3", null}, {2, {active, infinity}, null}}],
KeyList2 = [{{o, "Bucket1", "Key2", null}, {3, {active, infinity}, null}}],
{MergedKeyList, ListStatus, SN, _, _, _} = create_block(KeyList1,
KeyList2,
1),
?assertMatch(partial, ListStatus),
[H1|T1] = MergedKeyList,
?assertMatch(H1, {{o, "Bucket1", "Key1"}, {1, {active, infinity}, null}}),
?assertMatch(H1, {{o, "Bucket1", "Key1", null}, {1, {active, infinity}, null}}),
[H2|T2] = T1,
?assertMatch(H2, {{o, "Bucket1", "Key2"}, {3, {active, infinity}, null}}),
?assertMatch(T2, [{{o, "Bucket1", "Key3"}, {2, {active, infinity}, null}}]),
?assertMatch(H2, {{o, "Bucket1", "Key2", null}, {3, {active, infinity}, null}}),
?assertMatch(T2, [{{o, "Bucket1", "Key3", null}, {2, {active, infinity}, null}}]),
?assertMatch(SN, {1,3}).
dominate_create_block_test() ->
KeyList1 = [{{o, "Bucket1", "Key1"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key2"}, {2, {active, infinity}, null}}],
KeyList2 = [{{o, "Bucket1", "Key2"}, {3, {tomb, infinity}, null}}],
KeyList1 = [{{o, "Bucket1", "Key1", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key2", null}, {2, {active, infinity}, null}}],
KeyList2 = [{{o, "Bucket1", "Key2", null}, {3, {tomb, infinity}, null}}],
{MergedKeyList, ListStatus, SN, _, _, _} = create_block(KeyList1,
KeyList2,
1),
?assertMatch(partial, ListStatus),
[K1, K2] = MergedKeyList,
?assertMatch(K1, {{o, "Bucket1", "Key1"}, {1, {active, infinity}, null}}),
?assertMatch(K2, {{o, "Bucket1", "Key2"}, {3, {tomb, infinity}, null}}),
?assertMatch(K1, {{o, "Bucket1", "Key1", null}, {1, {active, infinity}, null}}),
?assertMatch(K2, {{o, "Bucket1", "Key2", null}, {3, {tomb, infinity}, null}}),
?assertMatch(SN, {1,3}).
sample_keylist() ->
KeyList1 = [{{o, "Bucket1", "Key1"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key3"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key5"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key7"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key1"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key3"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key5"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key7"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key9"}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key1"}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key3"}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key5"}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key7"}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key9"}, {1, {active, infinity}, null}},
{{o, "Bucket4", "Key1"}, {1, {active, infinity}, null}}],
KeyList2 = [{{o, "Bucket1", "Key2"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key4"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key6"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key8"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9a"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9b"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9c"}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9d"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key2"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key4"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key6"}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key8"}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key2"}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key4"}, {3, {active, infinity}, null}},
{{o, "Bucket3", "Key6"}, {2, {active, infinity}, null}},
{{o, "Bucket3", "Key8"}, {1, {active, infinity}, null}}],
KeyList1 = [{{o, "Bucket1", "Key1", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key3", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key5", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key7", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key1", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key3", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key5", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key7", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key9", null}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key1", null}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key3", null}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key5", null}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key7", null}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key9", null}, {1, {active, infinity}, null}},
{{o, "Bucket4", "Key1", null}, {1, {active, infinity}, null}}],
KeyList2 = [{{o, "Bucket1", "Key2", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key4", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key6", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key8", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9a", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9b", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9c", null}, {1, {active, infinity}, null}},
{{o, "Bucket1", "Key9d", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key2", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key4", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key6", null}, {1, {active, infinity}, null}},
{{o, "Bucket2", "Key8", null}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key2", null}, {1, {active, infinity}, null}},
{{o, "Bucket3", "Key4", null}, {3, {active, infinity}, null}},
{{o, "Bucket3", "Key6", null}, {2, {active, infinity}, null}},
{{o, "Bucket3", "Key8", null}, {1, {active, infinity}, null}}],
{KeyList1, KeyList2}.
alternating_create_block_test() ->
@ -1501,12 +1495,12 @@ alternating_create_block_test() ->
?assertMatch(BlockSize, 32),
?assertMatch(ListStatus, complete),
K1 = lists:nth(1, MergedKeyList),
?assertMatch(K1, {{o, "Bucket1", "Key1"}, {1, {active, infinity}, null}}),
?assertMatch(K1, {{o, "Bucket1", "Key1", null}, {1, {active, infinity}, null}}),
K11 = lists:nth(11, MergedKeyList),
?assertMatch(K11, {{o, "Bucket1", "Key9b"}, {1, {active, infinity}, null}}),
?assertMatch(K11, {{o, "Bucket1", "Key9b", null}, {1, {active, infinity}, null}}),
K32 = lists:nth(32, MergedKeyList),
?assertMatch(K32, {{o, "Bucket4", "Key1"}, {1, {active, infinity}, null}}),
HKey = {{o, "Bucket1", "Key0"}, {1, {active, infinity}, null}},
?assertMatch(K32, {{o, "Bucket4", "Key1", null}, {1, {active, infinity}, null}}),
HKey = {{o, "Bucket1", "Key0", null}, {1, {active, infinity}, null}},
{_, ListStatus2, _, _, _, _} = create_block([HKey|KeyList1], KeyList2, 1),
?assertMatch(ListStatus2, full).
@ -1565,17 +1559,17 @@ createslot_stage1_test() ->
{{LowKey, SegFilter, _SerialisedSlot, _LengthList},
{{LSN, HSN}, LastKey, Status},
KL1, KL2} = Out,
?assertMatch(LowKey, {o, "Bucket1", "Key1"}),
?assertMatch(LastKey, {o, "Bucket4", "Key1"}),
?assertMatch(LowKey, {o, "Bucket1", "Key1", null}),
?assertMatch(LastKey, {o, "Bucket4", "Key1", null}),
?assertMatch(Status, partial),
?assertMatch(KL1, []),
?assertMatch(KL2, []),
R0 = check_for_segments(serialise_segment_filter(SegFilter),
[hash_for_segmentid({keyonly, {o, "Bucket1", "Key1"}})],
[hash_for_segmentid({keyonly, {o, "Bucket1", "Key1", null}})],
true),
?assertMatch(R0, {maybe_present, [0]}),
R1 = check_for_segments(serialise_segment_filter(SegFilter),
[hash_for_segmentid({keyonly, {o, "Bucket1", "Key99"}})],
[hash_for_segmentid({keyonly, {o, "Bucket1", "Key99", null}})],
true),
?assertMatch(R1, not_present),
?assertMatch(LSN, 1),
@ -1605,25 +1599,29 @@ createslot_stage3_test() ->
Sum1 = lists:foldl(fun(X, Sum) -> Sum + X end, 0, LengthList),
Sum2 = byte_size(SerialisedSlot),
?assertMatch(Sum1, Sum2),
?assertMatch(LowKey, {o, "BucketSeq", "Key00000001"}),
?assertMatch(LastKey, {o, "BucketSeq", "Key00000128"}),
?assertMatch(LowKey, {o, "BucketSeq", "Key00000001", null}),
?assertMatch(LastKey, {o, "BucketSeq", "Key00000128", null}),
?assertMatch(KL1, []),
Rem = length(KL2),
?assertMatch(Rem, 72),
R0 = check_for_segments(serialise_segment_filter(SegFilter),
[hash_for_segmentid({keyonly, {o, "BucketSeq", "Key00000100"}})],
[hash_for_segmentid({keyonly,
{o, "BucketSeq", "Key00000100", null}})],
true),
?assertMatch(R0, {maybe_present, [3]}),
R1 = check_for_segments(serialise_segment_filter(SegFilter),
[hash_for_segmentid({keyonly, {o, "Bucket1", "Key99"}})],
[hash_for_segmentid({keyonly,
{o, "Bucket1", "Key99", null}})],
true),
?assertMatch(R1, not_present),
R2 = check_for_segments(serialise_segment_filter(SegFilter),
[hash_for_segmentid({keyonly, {o, "BucketSeq", "Key00000040"}})],
[hash_for_segmentid({keyonly,
{o, "BucketSeq", "Key00000040", null}})],
true),
?assertMatch(R2, {maybe_present, [1]}),
R3 = check_for_segments(serialise_segment_filter(SegFilter),
[hash_for_segmentid({keyonly, {o, "BucketSeq", "Key00000004"}})],
[hash_for_segmentid({keyonly,
{o, "BucketSeq", "Key00000004", null}})],
true),
?assertMatch(R3, {maybe_present, [0]}).
@ -1640,11 +1638,11 @@ writekeys_stage1_test() ->
fun testwrite_function/2),
{Handle, {_, PointerIndex}, SNExtremes, KeyExtremes} = FunOut,
?assertMatch(SNExtremes, {1,3}),
?assertMatch(KeyExtremes, {{o, "Bucket1", "Key1"},
{o, "Bucket4", "Key1"}}),
?assertMatch(KeyExtremes, {{o, "Bucket1", "Key1", null},
{o, "Bucket4", "Key1", null}}),
[TopIndex|[]] = PointerIndex,
{TopKey, _SegFilter, {LengthList, _Total}} = TopIndex,
?assertMatch(TopKey, {o, "Bucket1", "Key1"}),
?assertMatch(TopKey, {o, "Bucket1", "Key1", null}),
TotalLength = lists:foldl(fun(X, Acc) -> Acc + X end,
0, LengthList),
ActualLength = lists:foldl(fun(X, Acc) -> Acc + byte_size(X) end,
@ -1660,11 +1658,11 @@ initial_create_file_test() ->
{KL1, KL2} = sample_keylist(),
{Handle, FileMD} = create_file(Filename),
{UpdHandle, UpdFileMD, {[], []}} = complete_file(Handle, FileMD, KL1, KL2, 1),
Result1 = fetch_keyvalue(UpdHandle, UpdFileMD, {o, "Bucket1", "Key8"}),
Result1 = fetch_keyvalue(UpdHandle, UpdFileMD, {o, "Bucket1", "Key8", null}),
io:format("Result is ~w~n", [Result1]),
?assertMatch(Result1, {{o, "Bucket1", "Key8"},
?assertMatch(Result1, {{o, "Bucket1", "Key8", null},
{1, {active, infinity}, null}}),
Result2 = fetch_keyvalue(UpdHandle, UpdFileMD, {o, "Bucket1", "Key88"}),
Result2 = fetch_keyvalue(UpdHandle, UpdFileMD, {o, "Bucket1", "Key88", null}),
io:format("Result is ~w~n", [Result2]),
?assertMatch(Result2, not_present),
ok = file:close(UpdHandle),
@ -1699,7 +1697,9 @@ big_create_file_test() ->
SubList),
io:format("FailedFinds of ~w~n", [FailedFinds]),
?assertMatch(FailedFinds, 0),
Result3 = fetch_keyvalue(Handle, FileMD, {o, "Bucket1024", "Key1024Alt"}),
Result3 = fetch_keyvalue(Handle,
FileMD,
{o, "Bucket1024", "Key1024Alt", null}),
?assertMatch(Result3, not_present),
ok = file:close(Handle),
ok = file:delete(Filename).
@ -1708,35 +1708,46 @@ initial_iterator_test() ->
Filename = "../test/test2.sft",
{KL1, KL2} = sample_keylist(),
{Handle, FileMD} = create_file(Filename),
{UpdHandle, UpdFileMD, {[], []}} = complete_file(Handle, FileMD, KL1, KL2, 1),
{UpdHandle, UpdFileMD, {[], []}} = complete_file(Handle,
FileMD,
KL1,
KL2,
1),
Result1 = fetch_range_keysonly(UpdHandle, UpdFileMD,
{o, "Bucket1", "Key8"},
{o, "Bucket1", "Key9d"}),
{o, "Bucket1", "Key8", null},
{o, "Bucket1", "Key9d", null}),
io:format("Result returned of ~w~n", [Result1]),
?assertMatch(Result1, {complete, [{{o, "Bucket1", "Key8"}, 1},
{{o, "Bucket1", "Key9"}, 1},
{{o, "Bucket1", "Key9a"}, 1},
{{o, "Bucket1", "Key9b"}, 1},
{{o, "Bucket1", "Key9c"}, 1},
{{o, "Bucket1", "Key9d"}, 1}]}),
?assertMatch({complete,
[{{o, "Bucket1", "Key8", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9a", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9b", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9c", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9d", null}, 1, {active, infinity}}
]},
Result1),
Result2 = fetch_range_keysonly(UpdHandle, UpdFileMD,
{o, "Bucket1", "Key8"},
{o, "Bucket1", "Key9b"}),
?assertMatch(Result2, {complete, [{{o, "Bucket1", "Key8"}, 1},
{{o, "Bucket1", "Key9"}, 1},
{{o, "Bucket1", "Key9a"}, 1},
{{o, "Bucket1", "Key9b"}, 1}]}),
{o, "Bucket1", "Key8", null},
{o, "Bucket1", "Key9b", null}),
?assertMatch({complete,
[{{o, "Bucket1", "Key8", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9a", null}, 1, {active, infinity}},
{{o, "Bucket1", "Key9b", null}, 1, {active, infinity}}
]},
Result2),
Result3 = fetch_range_keysonly(UpdHandle, UpdFileMD,
{o, "Bucket3", "Key4"},
{o, "Bucket3", "Key4", null},
all),
{partial, RL3, _} = Result3,
?assertMatch(RL3, [{{o, "Bucket3", "Key4"}, 3},
{{o, "Bucket3", "Key5"}, 1},
{{o, "Bucket3", "Key6"}, 2},
{{o, "Bucket3", "Key7"}, 1},
{{o, "Bucket3", "Key8"}, 1},
{{o, "Bucket3", "Key9"}, 1},
{{o, "Bucket4", "Key1"}, 1}]),
?assertMatch([{{o, "Bucket3", "Key4", null}, 3, {active, infinity}},
{{o, "Bucket3", "Key5", null}, 1, {active, infinity}},
{{o, "Bucket3", "Key6", null}, 2, {active, infinity}},
{{o, "Bucket3", "Key7", null}, 1, {active, infinity}},
{{o, "Bucket3", "Key8", null}, 1, {active, infinity}},
{{o, "Bucket3", "Key9", null}, 1, {active, infinity}},
{{o, "Bucket4", "Key1", null}, 1, {active, infinity}}],
RL3),
ok = file:close(UpdHandle),
ok = file:delete(Filename).
@ -1748,22 +1759,26 @@ big_iterator_test() ->
InitFileMD,
KL1, KL2, 1),
io:format("Remainder lengths are ~w and ~w ~n", [length(KL1Rem), length(KL2Rem)]),
{complete, Result1} = fetch_range_keysonly(Handle, FileMD, {o, "Bucket0000", "Key0000"},
{o, "Bucket9999", "Key9999"},
{complete, Result1} = fetch_range_keysonly(Handle,
FileMD,
{o, "Bucket0000", "Key0000", null},
{o, "Bucket9999", "Key9999", null},
256),
NumFoundKeys1 = length(Result1),
NumAddedKeys = 10000 - length(KL1Rem),
?assertMatch(NumFoundKeys1, NumAddedKeys),
{partial, Result2, _} = fetch_range_keysonly(Handle, FileMD, {o, "Bucket0000", "Key0000"},
{o, "Bucket9999", "Key9999"},
{partial, Result2, _} = fetch_range_keysonly(Handle,
FileMD,
{o, "Bucket0000", "Key0000", null},
{o, "Bucket9999", "Key9999", null},
32),
NumFoundKeys2 = length(Result2),
?assertMatch(NumFoundKeys2, 32 * 128),
{partial, Result3, _} = fetch_range_keysonly(Handle, FileMD, {o, "Bucket0000", "Key0000"},
{o, "Bucket9999", "Key9999"},
?assertMatch(32 * 128, length(Result2)),
{partial, Result3, _} = fetch_range_keysonly(Handle,
FileMD,
{o, "Bucket0000", "Key0000", null},
{o, "Bucket9999", "Key9999", null},
4),
NumFoundKeys3 = length(Result3),
?assertMatch(NumFoundKeys3, 4 * 128),
?assertMatch(4 * 128, length(Result3)),
ok = file:close(Handle),
ok = file:delete(Filename).

View file

@ -7,9 +7,9 @@
journal_compaction/1,
fetchput_snapshot/1]).
all() -> [simple_put_fetch_head,
many_put_fetch_head,
journal_compaction,
all() -> [% simple_put_fetch_head,
% many_put_fetch_head,
% journal_compaction,
fetchput_snapshot].
@ -203,6 +203,15 @@ fetchput_snapshot(_Config) ->
{ok, FNsC} = file:list_dir(RootPath ++ "/ledger/ledger_files"),
true = length(FNsB) > length(FNsA),
true = length(FNsB) > length(FNsC),
{B1Size, B1Count} = check_bucket_stats(Bookie2, "Bucket1"),
true = B1Size > 0,
true = B1Count == 1,
{B1Size, B1Count} = check_bucket_stats(Bookie2, "Bucket1"),
{BSize, BCount} = check_bucket_stats(Bookie2, "Bucket"),
true = BSize > 0,
true = BCount == 140000,
ok = leveled_bookie:book_close(Bookie2),
reset_filestructure().
@ -217,6 +226,21 @@ reset_filestructure() ->
RootPath.
check_bucket_stats(Bookie, Bucket) ->
FoldSW1 = os:timestamp(),
io:format("Checking bucket size~n"),
{async, Folder1} = leveled_bookie:book_returnfolder(Bookie,
{bucket_stats,
Bucket}),
{B1Size, B1Count} = Folder1(),
io:format("Bucket fold completed in ~w microseconds~n",
[timer:now_diff(os:timestamp(), FoldSW1)]),
io:format("Bucket ~w has size ~w and count ~w~n",
[Bucket, B1Size, B1Count]),
{B1Size, B1Count}.
check_bookie_forlist(Bookie, ChkList) ->
check_bookie_forlist(Bookie, ChkList, false).