41#include <condition_variable>
53 "Do not store real data with this version of RNTuple!";
54 fCompressor = std::make_unique<RNTupleCompressor>();
80 fWriter = std::unique_ptr<Internal::RNTupleFileWriter>(
90 unsigned char *serializedHeader, std::uint32_t
length)
92 auto zipBuffer = std::make_unique<unsigned char[]>(
length);
93 auto szZipHeader = fCompressor->Zip(serializedHeader,
length, GetWriteOptions().GetCompression(),
95 fWriter->WriteNTupleHeader(zipBuffer.get(), szZipHeader,
length);
103 std::uint64_t offsetData;
106 offsetData = fWriter->WriteBlob(sealedPage.
fBuffer, sealedPage.
fSize, bytesPacked);
110 result.fPosition = offsetData;
112 fCounters->fNPageCommitted.Inc();
113 fCounters->fSzWritePayload.Add(sealedPage.
fSize);
114 fNBytesCurrentCluster += sealedPage.
fSize;
126 sealedPage = SealPage(page, *element, GetWriteOptions().GetCompression());
130 return WriteSealedPage(sealedPage, element->GetPackedSize(page.
GetNElements()));
138 fDescriptorBuilder.GetDescriptor().GetColumnDescriptor(physicalColumnId).GetModel().GetType());
139 const auto bytesPacked = (bitsOnStorage * sealedPage.
fNElements + 7) / 8;
141 return WriteSealedPage(sealedPage, bytesPacked);
148 auto result = fNBytesCurrentCluster;
149 fNBytesCurrentCluster = 0;
157 auto bufPageListZip = std::make_unique<unsigned char[]>(
length);
158 auto szPageListZip = fCompressor->Zip(serializedPageList,
length, GetWriteOptions().GetCompression(),
162 result.fBytesOnStorage = szPageListZip;
163 result.fPosition = fWriter->WriteBlob(bufPageListZip.get(), szPageListZip,
length);
169 auto bufFooterZip = std::make_unique<unsigned char[]>(
length);
170 auto szFooterZip = fCompressor->Zip(serializedFooter,
length, GetWriteOptions().GetCompression(),
172 fWriter->WriteNTupleFooter(bufFooterZip.get(), szFooterZip,
length);
182 return fPageAllocator->NewPage(columnHandle.
fPhysicalId, elementSize, nElements);
187 fPageAllocator->DeletePage(page);
196 fPagePool(std::make_shared<
RPagePool>()),
197 fClusterPool(std::make_unique<
RClusterPool>(*this, options.GetClusterBunchSize()))
215 fDescriptorBuilder.SetOnDiskHeaderSize(anchor.
fNBytesHeader);
216 auto buffer = std::make_unique<unsigned char[]>(anchor.
fLenHeader);
217 auto zipBuffer = std::make_unique<unsigned char[]>(anchor.
fNBytesHeader);
222 fDescriptorBuilder.AddToOnDiskFooterSize(anchor.
fNBytesFooter);
223 buffer = std::make_unique<unsigned char[]>(anchor.
fLenFooter);
224 zipBuffer = std::make_unique<unsigned char[]>(anchor.
fNBytesFooter);
230std::unique_ptr<ROOT::Experimental::Detail::RPageSourceFile>
234 auto pageSource = std::make_unique<RPageSourceFile>(
"", path, options);
235 pageSource->InitDescriptor(anchor);
236 pageSource->fNTupleName = pageSource->fDescriptorBuilder.GetDescriptor().GetName();
247 if (fDescriptorBuilder.GetDescriptor().GetOnDiskHeaderSize() == 0) {
248 auto anchor = fReader.GetNTuple(fNTupleName).Unwrap();
249 InitDescriptor(anchor);
252 auto ntplDesc = fDescriptorBuilder.MoveDescriptor();
254 for (
const auto &cgDesc : ntplDesc.GetClusterGroupIterable()) {
255 auto buffer = std::make_unique<unsigned char[]>(cgDesc.GetPageListLength());
256 auto zipBuffer = std::make_unique<unsigned char[]>(cgDesc.GetPageListLocator().fBytesOnStorage);
257 fReader.ReadBuffer(zipBuffer.get(), cgDesc.GetPageListLocator().fBytesOnStorage,
258 cgDesc.GetPageListLocator().GetPosition<std::uint64_t>());
259 fDecompressor->Unzip(zipBuffer.get(), cgDesc.GetPageListLocator().fBytesOnStorage, cgDesc.GetPageListLength(),
264 for (std::size_t i = 0; i < clusters.size(); ++i) {
265 ntplDesc.AddClusterDetails(clusters[i].AddDeferredColumnRanges(ntplDesc).MoveDescriptor().Unwrap());
280 auto descriptorGuard = GetSharedDescriptorGuard();
281 const auto &clusterDescriptor = descriptorGuard->GetClusterDescriptor(clusterId);
282 pageInfo = clusterDescriptor.GetPageRange(physicalColumnId).Find(clusterIndex.
GetIndex());
286 sealedPage.
fSize = bytesOnStorage;
291 fReader.ReadBuffer(
const_cast<void *
>(sealedPage.
fBuffer), bytesOnStorage,
304 const auto clusterId = clusterInfo.
fClusterId;
305 const auto pageInfo = clusterInfo.
fPageInfo;
308 const auto elementSize = element->
GetSize();
309 const auto bytesOnStorage = pageInfo.fLocator.fBytesOnStorage;
311 const void *sealedPageBuffer =
nullptr;
312 std::unique_ptr<unsigned char []> directReadBuffer;
316 pageZero.GrowUnchecked(pageInfo.fNElements);
317 pageZero.SetWindow(clusterInfo.
fColumnOffset + pageInfo.fFirstInPage,
319 fPagePool->RegisterPage(pageZero,
RPageDeleter([](
const RPage &,
void *) {},
nullptr));
324 directReadBuffer = std::unique_ptr<unsigned char[]>(
new unsigned char[bytesOnStorage]);
325 fReader.ReadBuffer(directReadBuffer.get(), bytesOnStorage, pageInfo.fLocator.GetPosition<std::uint64_t>());
326 fCounters->fNPageLoaded.Inc();
327 fCounters->fNRead.Inc();
328 fCounters->fSzReadPayload.Add(bytesOnStorage);
329 sealedPageBuffer = directReadBuffer.get();
331 if (!fCurrentCluster || (fCurrentCluster->GetId() != clusterId) || !fCurrentCluster->ContainsColumn(columnId))
332 fCurrentCluster = fClusterPool->GetCluster(clusterId, fActivePhysicalColumns.ToColumnSet());
333 R__ASSERT(fCurrentCluster->ContainsColumn(columnId));
335 auto cachedPage = fPagePool->GetPage(columnId,
RClusterIndex(clusterId, idxInCluster));
336 if (!cachedPage.IsNull())
340 auto onDiskPage = fCurrentCluster->GetOnDiskPage(key);
341 R__ASSERT(onDiskPage && (bytesOnStorage == onDiskPage->GetSize()));
342 sealedPageBuffer = onDiskPage->GetAddress();
348 newPage = UnsealPage({sealedPageBuffer, bytesOnStorage, pageInfo.fNElements}, *element, columnId);
349 fCounters->fSzUnzip.Add(elementSize * pageInfo.fNElements);
354 fPagePool->RegisterPage(
357 fCounters->fNPagePopulated.Inc();
366 auto cachedPage = fPagePool->GetPage(columnId, globalIndex);
367 if (!cachedPage.IsNull())
370 std::uint64_t idxInCluster;
373 auto descriptorGuard = GetSharedDescriptorGuard();
374 clusterInfo.
fClusterId = descriptorGuard->FindClusterId(columnId, globalIndex);
377 const auto &clusterDescriptor = descriptorGuard->GetClusterDescriptor(clusterInfo.
fClusterId);
378 clusterInfo.
fColumnOffset = clusterDescriptor.GetColumnRange(columnId).fFirstElementIndex;
381 clusterInfo.
fPageInfo = clusterDescriptor.GetPageRange(columnId).Find(idxInCluster);
384 return PopulatePageFromCluster(columnHandle, clusterInfo, idxInCluster);
392 const auto idxInCluster = clusterIndex.
GetIndex();
394 auto cachedPage = fPagePool->GetPage(columnId, clusterIndex);
395 if (!cachedPage.IsNull())
401 auto descriptorGuard = GetSharedDescriptorGuard();
402 const auto &clusterDescriptor = descriptorGuard->GetClusterDescriptor(clusterId);
404 clusterInfo.
fColumnOffset = clusterDescriptor.GetColumnRange(columnId).fFirstElementIndex;
405 clusterInfo.
fPageInfo = clusterDescriptor.GetPageRange(columnId).Find(idxInCluster);
408 return PopulatePageFromCluster(columnHandle, clusterInfo, idxInCluster);
413 fPagePool->ReturnPage(page);
421 return std::unique_ptr<RPageSourceFile>(
clone);
424std::unique_ptr<ROOT::Experimental::Detail::RCluster>
427 std::vector<ROOT::Internal::RRawFile::RIOVec> &readRequests)
429 struct ROnDiskPageLocator {
432 std::uint64_t fOffset = 0;
433 std::uint64_t
fSize = 0;
434 std::size_t fBufPos = 0;
437 std::vector<ROnDiskPageLocator> onDiskPages;
439 auto pageZeroMap = std::make_unique<ROnDiskPageMap>();
440 PrepareLoadCluster(clusterKey, *pageZeroMap,
443 const auto &pageLocator = pageInfo.
fLocator;
445 onDiskPages.push_back({physicalColumnId, pageNo, pageLocator.GetPosition<std::uint64_t>(),
446 pageLocator.fBytesOnStorage, 0});
450 std::sort(onDiskPages.begin(), onDiskPages.end(),
451 [](
const ROnDiskPageLocator &
a,
const ROnDiskPageLocator &
b) {return a.fOffset < b.fOffset;});
460 float maxOverhead = 0.25 * float(activeSize);
461 std::vector<std::size_t> gaps;
462 for (
unsigned i = 1; i < onDiskPages.size(); ++i) {
463 gaps.emplace_back(onDiskPages[i].fOffset - (onDiskPages[i-1].
fSize + onDiskPages[i-1].fOffset));
465 std::sort(gaps.begin(), gaps.end());
466 std::size_t gapCut = 0;
467 std::size_t currentGap = 0;
469 for (
auto g : gaps) {
470 if (
g != currentGap) {
475 if (szExtra > maxOverhead)
483 const auto currentReadRequestIdx = readRequests.size();
486 std::size_t szPayload = 0;
487 std::size_t szOverhead = 0;
488 for (
auto &s : onDiskPages) {
492 auto overhead = s.fOffset - readUpTo;
493 szPayload += s.fSize;
494 if (overhead <= gapCut) {
495 szOverhead += overhead;
496 s.fBufPos =
reinterpret_cast<intptr_t
>(req.
fBuffer) + req.
fSize + overhead;
497 req.
fSize += overhead + s.fSize;
503 readRequests.emplace_back(req);
506 s.fBufPos =
reinterpret_cast<intptr_t
>(req.
fBuffer);
511 readRequests.emplace_back(req);
512 fCounters->fSzReadPayload.Add(szPayload);
513 fCounters->fSzReadOverhead.Add(szOverhead);
516 auto buffer =
new unsigned char[
reinterpret_cast<intptr_t
>(req.
fBuffer) + req.
fSize];
517 auto pageMap = std::make_unique<ROnDiskPageMapHeap>(std::unique_ptr<
unsigned char []>(buffer));
518 for (
const auto &s : onDiskPages) {
520 pageMap->Register(key,
ROnDiskPage(buffer + s.fBufPos, s.fSize));
522 fCounters->fNPageLoaded.Add(onDiskPages.size());
523 for (
auto i = currentReadRequestIdx; i < readRequests.size(); ++i) {
524 readRequests[i].fBuffer = buffer +
reinterpret_cast<intptr_t
>(readRequests[i].fBuffer);
527 auto cluster = std::make_unique<RCluster>(clusterKey.
fClusterId);
528 cluster->Adopt(std::move(pageMap));
529 cluster->Adopt(std::move(pageZeroMap));
531 cluster->SetColumnAvailable(colId);
535std::vector<std::unique_ptr<ROOT::Experimental::Detail::RCluster>>
538 fCounters->fNClusterLoaded.Add(clusterKeys.size());
540 std::vector<std::unique_ptr<ROOT::Experimental::Detail::RCluster>> clusters;
541 std::vector<ROOT::Internal::RRawFile::RIOVec> readRequests;
543 for (
auto key: clusterKeys) {
544 clusters.emplace_back(PrepareSingleCluster(key, readRequests));
547 auto nReqs = readRequests.size();
550 fFile->ReadV(&readRequests[0], nReqs);
552 fCounters->fNReadV.Inc();
553 fCounters->fNRead.Add(nReqs);
562 fTaskScheduler->Reset();
564 const auto clusterId = cluster->
GetId();
565 auto descriptorGuard = GetSharedDescriptorGuard();
566 const auto &clusterDescriptor = descriptorGuard->GetClusterDescriptor(clusterId);
568 std::vector<std::unique_ptr<RColumnElementBase>> allElements;
571 for (
const auto columnId : columnsInCluster) {
572 const auto &columnDesc = descriptorGuard->GetColumnDescriptor(columnId);
576 const auto &pageRange = clusterDescriptor.GetPageRange(columnId);
577 std::uint64_t pageNo = 0;
578 std::uint64_t firstInPage = 0;
579 for (
const auto &pi : pageRange.fPageInfos) {
582 R__ASSERT(onDiskPage && (onDiskPage->GetSize() == pi.fLocator.fBytesOnStorage));
584 auto taskFunc = [
this, columnId, clusterId, firstInPage, onDiskPage, element = allElements.back().get(),
585 nElements = pi.fNElements,
586 indexOffset = clusterDescriptor.GetColumnRange(columnId).fFirstElementIndex]() {
587 auto newPage = UnsealPage({onDiskPage->GetAddress(), onDiskPage->GetSize(), nElements}, *element, columnId);
588 fCounters->fSzUnzip.Add(element->GetSize() * nElements);
591 fPagePool->PreloadPage(
597 fTaskScheduler->AddTask(taskFunc);
599 firstInPage += pi.fNElements;
606 fTaskScheduler->Wait();
#define R__FAIL(msg)
Short-hand to return an RResult<T> in an error state; the RError is implicitly converted into RResult...
#define R__LOG_WARNING(...)
TObject * clone(const char *newname) const override
Option_t Option_t TPoint TPoint const char GetTextMagnitude GetFillStyle GetLineColor GetLineWidth GetMarkerStyle GetTextAlign GetTextColor GetTextSize void char Point_t Rectangle_t WindowAttributes_t Float_t Float_t Float_t Int_t Int_t UInt_t UInt_t Rectangle_t result
Option_t Option_t TPoint TPoint const char GetTextMagnitude GetFillStyle GetLineColor GetLineWidth GetMarkerStyle GetTextAlign GetTextColor GetTextSize void char Point_t Rectangle_t WindowAttributes_t Float_t Float_t Float_t Int_t Int_t UInt_t UInt_t Rectangle_t Int_t Int_t Window_t TString Int_t GCValues_t GetPrimarySelectionOwner GetDisplay GetScreen GetColormap GetNativeEvent const char const char dpyName wid window const char font_name cursor keysym reg const char only_if_exist regb h Point_t winding char text const char depth char const char Int_t count const char ColorStruct_t color const char Pixmap_t Pixmap_t PictureAttributes_t attr const char char ret_data h unsigned char height h length
Managed a set of clusters containing compressed and packed pages.
An in-memory subset of the packed and compressed pages of a cluster.
size_t GetNOnDiskPages() const
DescriptorId_t GetId() const
const ColumnSet_t & GetAvailPhysicalColumns() const
const ROnDiskPage * GetOnDiskPage(const ROnDiskPage::Key &key) const
virtual std::size_t GetBitsOnStorage() const
std::size_t GetSize() const
static std::unique_ptr< RColumnElementBase > Generate(EColumnType type)
If CppT == void, use the default C++ type for the given column type.
RColumnElementBase * GetElement() const
static Writer_t MakeMemCopyWriter(unsigned char *dest)
Record wall time and CPU time between construction and destruction.
A page as being stored on disk, that is packed and compressed.
Uses standard C++ memory allocation for the column data pages.
static void DeletePage(const RPage &page)
Releases the memory pointed to by page and resets the page's information.
A closure that can free the memory associated with a mapped page.
A thread-safe cache of column pages.
Storage provider that write ntuple pages into a file.
~RPageSinkFile() override
RPage ReservePage(ColumnHandle_t columnHandle, std::size_t nElements) final
Get a new, empty page for the given column that can be filled with up to nElements.
RNTupleLocator CommitPageImpl(ColumnHandle_t columnHandle, const RPage &page) final
std::uint64_t CommitClusterImpl(NTupleSize_t nEntries) final
Returns the number of bytes written to storage (excluding metadata)
void CreateImpl(const RNTupleModel &model, unsigned char *serializedHeader, std::uint32_t length) final
RPageSinkFile(std::string_view ntupleName, const RNTupleWriteOptions &options)
void ReleasePage(RPage &page) final
Every page store needs to be able to free pages it handed out.
void CommitDatasetImpl(unsigned char *serializedFooter, std::uint32_t length) final
std::unique_ptr< Internal::RNTupleFileWriter > fWriter
RNTupleLocator CommitSealedPageImpl(DescriptorId_t physicalColumnId, const RPageStorage::RSealedPage &sealedPage) final
RNTupleLocator WriteSealedPage(const RPageStorage::RSealedPage &sealedPage, std::size_t bytesPacked)
RNTupleLocator CommitClusterGroupImpl(unsigned char *serializedPageList, std::uint32_t length) final
Returns the locator of the page list envelope of the given buffer that contains the serialized page l...
Abstract interface to write data into an ntuple.
void EnableDefaultMetrics(const std::string &prefix)
Enables the default set of metrics provided by RPageSink.
std::unique_ptr< RNTupleCompressor > fCompressor
Helper to zip pages and header/footer; includes a 16MB (kMAXZIPBUF) zip buffer.
Storage provider that reads ntuple pages from a file.
void LoadSealedPage(DescriptorId_t physicalColumnId, const RClusterIndex &clusterIndex, RSealedPage &sealedPage) final
Read the packed and compressed bytes of a page into the memory buffer provided by selaedPage.
std::vector< std::unique_ptr< RCluster > > LoadClusters(std::span< RCluster::RKey > clusterKeys) final
Populates all the pages of the given cluster ids and columns; it is possible that some columns do not...
void InitDescriptor(const Internal::RFileNTupleAnchor &anchor)
Deserialized header and footer into a minimal descriptor held by fDescriptorBuilder.
Internal::RMiniFileReader fReader
Takes the fFile to read ntuple blobs from it.
void ReleasePage(RPage &page) final
Every page store needs to be able to free pages it handed out.
static std::unique_ptr< RPageSourceFile > CreateFromAnchor(const Internal::RFileNTupleAnchor &anchor, std::string_view path, const RNTupleReadOptions &options)
Used from the RNTuple class to build a datasource if the anchor is already available.
RNTupleDescriptor AttachImpl() final
std::unique_ptr< RCluster > PrepareSingleCluster(const RCluster::RKey &clusterKey, std::vector< ROOT::Internal::RRawFile::RIOVec > &readRequests)
Helper function for LoadClusters: it prepares the memory buffer (page map) and the read requests for ...
std::unique_ptr< RPageSource > Clone() const final
The cloned page source creates a new raw file and reader and opens its own file descriptor to the dat...
void UnzipClusterImpl(RCluster *cluster) final
~RPageSourceFile() override
RPageSourceFile(std::string_view ntupleName, const RNTupleReadOptions &options)
std::unique_ptr< ROOT::Internal::RRawFile > fFile
An RRawFile is used to request the necessary byte ranges from a local or a remote file.
RPage PopulatePage(ColumnHandle_t columnHandle, NTupleSize_t globalIndex) final
Allocates and fills a page that contains the index-th element.
RPage PopulatePageFromCluster(ColumnHandle_t columnHandle, const RClusterInfo &clusterInfo, ClusterSize_t::ValueType idxInCluster)
Abstract interface to read data from an ntuple.
void EnableDefaultMetrics(const std::string &prefix)
Enables the default set of metrics provided by RPageSource.
std::unique_ptr< RNTupleDecompressor > fDecompressor
Helper to unzip pages and header/footer; comprises a 16MB (kMAXZIPBUF) unzip buffer.
Stores information about the cluster in which this page resides.
A page is a slice of a column that is mapped into memory.
std::uint32_t GetNBytes() const
The space taken by column elements in the buffer.
static RPage MakePageZero(ColumnId_t columnId, ClusterSize_t::ValueType elementSize)
Make a 'zero' page for column columnId (that is comprised of 0x00 bytes only).
std::uint32_t GetNElements() const
void SetWindow(const NTupleSize_t rangeFirst, const RClusterInfo &clusterInfo)
Seek the page to a certain position of the column.
static const void * GetPageZeroBuffer()
Return a pointer to the page zero buffer used if there is no on-disk data for a particular deferred c...
Read RNTuple data blocks from a TFile container, provided by a RRawFile.
static RNTupleFileWriter * Append(std::string_view ntupleName, TFile &file)
Add a new RNTuple identified by ntupleName to the existing TFile.
static RNTupleFileWriter * Recreate(std::string_view ntupleName, std::string_view path, int defaultCompression, ENTupleContainerFormat containerFormat)
Create or truncate the local file given by path with the new empty RNTuple identified by ntupleName.
static RResult< void > DeserializePageListV1(const void *buffer, std::uint32_t bufSize, std::vector< RClusterDescriptorBuilder > &clusters)
static RResult< void > DeserializeFooterV1(const void *buffer, std::uint32_t bufSize, RNTupleDescriptorBuilder &descBuilder)
static RResult< void > DeserializeHeaderV1(const void *buffer, std::uint32_t bufSize, RNTupleDescriptorBuilder &descBuilder)
static std::vector< RClusterDescriptorBuilder > GetClusterSummaries(const RNTupleDescriptor &ntplDesc, DescriptorId_t clusterGroupId)
Used to prepare the cluster descriptor builders when loading the page locations for a certain cluster...
Addresses a column element or field item relative to a particular cluster, instead of a global NTuple...
DescriptorId_t GetClusterId() const
ClusterSize_t::ValueType GetIndex() const
Base class for all ROOT issued exceptions.
The on-storage meta-data of an ntuple.
The RNTupleModel encapulates the schema of an ntuple.
Common user-tunable settings for reading ntuples.
Common user-tunable settings for storing ntuples.
int GetCompression() const
ENTupleContainerFormat GetContainerFormat() const
static std::unique_ptr< RRawFile > Create(std::string_view url, ROptions options=ROptions())
Factory method that returns a suitable concrete implementation according to the transport in the url.
A ROOT file is composed of a header, followed by consecutive data records (TKey instances) with a wel...
virtual TObject * Clone(const char *newname="") const
Make a clone of an object using the Streamer facility.
RLogChannel & NTupleLog()
Log channel for RNTuple diagnostics.
std::uint64_t NTupleSize_t
Integer type long enough to hold the maximum number of entries in a column.
std::uint64_t DescriptorId_t
Distriniguishes elements of the same type within a descriptor, e.g. different fields.
constexpr DescriptorId_t kInvalidDescriptorId
The identifiers that specifies the content of a (partial) cluster.
DescriptorId_t fClusterId
ColumnSet_t fPhysicalColumnSet
On-disk pages within a page source are identified by the column and page number.
Summarizes cluster-level information that are necessary to populate a certain page.
DescriptorId_t fClusterId
RClusterDescriptor::RPageRange::RPageInfoExtended fPageInfo
Location of the page on disk.
std::uint64_t fColumnOffset
The first element number of the page's column in the given cluster.
DescriptorId_t fPhysicalId
A sealed page contains the bytes of a page as written to storage (packed & compressed).
Entry point for an RNTuple in a ROOT file.
std::uint32_t fNBytesFooter
The size of the compressed ntuple footer.
std::uint64_t fSeekFooter
The file offset of the footer excluding the TKey part.
std::uint32_t fNBytesHeader
The size of the compressed ntuple header.
std::uint32_t fLenFooter
The size of the uncompressed ntuple footer.
std::uint64_t fSeekHeader
The file offset of the header excluding the TKey part.
std::uint32_t fLenHeader
The size of the uncompressed ntuple header.
Generic information about the physical location of data.
ELocatorType fType
For non-disk locators, the value for the Type field.
std::uint32_t fBytesOnStorage
const T & GetPosition() const
Used for vector reads from multiple offsets into multiple buffers.
std::size_t fSize
The number of desired bytes.
void * fBuffer
The destination for reading.
std::uint64_t fOffset
The file offset.