Merge remote-tracking branch 'origin/3.0' into feature/config

This commit is contained in:
Shengliang Guan 2022-02-28 16:35:36 +08:00
commit e5606ccf38
33 changed files with 3166 additions and 1360 deletions

View File

@ -589,7 +589,7 @@ SOperatorInfo* createTableSeqScanOperatorInfo(void* pTsdbQueryHandle, SQueryRunt
SOperatorInfo* createAggregateOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput); SOperatorInfo* createAggregateOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput);
SOperatorInfo* createProjectOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput); SOperatorInfo* createProjectOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput);
SOperatorInfo* createLimitOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream); SOperatorInfo* createLimitOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream);
SOperatorInfo* createTimeIntervalOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput); SOperatorInfo* createIntervalOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput);
SOperatorInfo* createAllTimeIntervalOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput); SOperatorInfo* createAllTimeIntervalOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput);
SOperatorInfo* createSWindowOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput); SOperatorInfo* createSWindowOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput);
SOperatorInfo* createFillOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput, bool multigroupResult); SOperatorInfo* createFillOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput, bool multigroupResult);

View File

@ -2166,7 +2166,7 @@ static int32_t setupQueryRuntimeEnv(SQueryRuntimeEnv *pRuntimeEnv, int32_t numOf
} }
case OP_TimeWindow: { case OP_TimeWindow: {
pRuntimeEnv->proot = pRuntimeEnv->proot =
createTimeIntervalOperatorInfo(pRuntimeEnv, pRuntimeEnv->proot, pQueryAttr->pExpr1, pQueryAttr->numOfOutput); createIntervalOperatorInfo(pRuntimeEnv, pRuntimeEnv->proot, pQueryAttr->pExpr1, pQueryAttr->numOfOutput);
int32_t opType = pRuntimeEnv->proot->upstream[0]->operatorType; int32_t opType = pRuntimeEnv->proot->upstream[0]->operatorType;
if (opType != OP_DummyInput && opType != OP_Join) { if (opType != OP_DummyInput && opType != OP_Join) {
setTableScanFilterOperatorInfo(pRuntimeEnv->proot->upstream[0]->info, pRuntimeEnv->proot); setTableScanFilterOperatorInfo(pRuntimeEnv->proot->upstream[0]->info, pRuntimeEnv->proot);
@ -6756,7 +6756,7 @@ SOperatorInfo* createLimitOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorI
return pOperator; return pOperator;
} }
SOperatorInfo* createTimeIntervalOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput) { SOperatorInfo* createIntervalOperatorInfo(SQueryRuntimeEnv* pRuntimeEnv, SOperatorInfo* upstream, SExprInfo* pExpr, int32_t numOfOutput) {
STableIntervalOperatorInfo* pInfo = calloc(1, sizeof(STableIntervalOperatorInfo)); STableIntervalOperatorInfo* pInfo = calloc(1, sizeof(STableIntervalOperatorInfo));
pInfo->pCtx = createSQLFunctionCtx(pRuntimeEnv, pExpr, numOfOutput, &pInfo->rowCellInfoOffset); pInfo->pCtx = createSQLFunctionCtx(pRuntimeEnv, pExpr, numOfOutput, &pInfo->rowCellInfoOffset);

View File

@ -18,9 +18,6 @@ typedef struct SBlockOrderInfo {
int32_t order; int32_t order;
int32_t colIndex; int32_t colIndex;
SColumnInfoData *pColData; SColumnInfoData *pColData;
// int32_t type;
// int32_t bytes;
// bool hasNull;
} SBlockOrderInfo; } SBlockOrderInfo;
int taosGetFqdnPortFromEp(const char *ep, SEp *pEp); int taosGetFqdnPortFromEp(const char *ep, SEp *pEp);
@ -67,16 +64,16 @@ static FORCE_INLINE bool colDataIsNull(const SColumnInfoData* pColumnInfoData, u
} }
} }
#define colDataGet(p1_, r_) \ #define colDataGetData(p1_, r_) \
((IS_VAR_DATA_TYPE((p1_)->info.type)) ? ((p1_)->pData + (p1_)->varmeta.offset[(r_)]) \ ((IS_VAR_DATA_TYPE((p1_)->info.type)) ? (p1_)->pData + (p1_)->varmeta.offset[(r_)] \
: ((p1_)->pData + ((r_) * (p1_)->info.bytes))) : (p1_)->pData + ((r_) * (p1_)->info.bytes))
int32_t colDataAppend(SColumnInfoData* pColumnInfoData, uint32_t currentRow, const char* pData, bool isNull); int32_t colDataAppend(SColumnInfoData* pColumnInfoData, uint32_t currentRow, const char* pData, bool isNull);
int32_t colDataMergeCol(SColumnInfoData* pColumnInfoData, uint32_t numOfRow1, const SColumnInfoData* pSource, uint32_t numOfRow2); int32_t colDataMergeCol(SColumnInfoData* pColumnInfoData, uint32_t numOfRow1, const SColumnInfoData* pSource, uint32_t numOfRow2);
int32_t blockDataUpdateTsWindow(SSDataBlock* pDataBlock); int32_t blockDataUpdateTsWindow(SSDataBlock* pDataBlock);
int32_t colDataGetSize(const SColumnInfoData* pColumnInfoData, int32_t numOfRows); int32_t colDataGetLength(const SColumnInfoData* pColumnInfoData, int32_t numOfRows);
void colDataTrim(SColumnInfoData* pColumnInfoData); void colDataTrim(SColumnInfoData* pColumnInfoData);
size_t colDataGetNumOfCols(const SSDataBlock* pBlock); size_t colDataGetNumOfCols(const SSDataBlock* pBlock);
size_t colDataGetNumOfRows(const SSDataBlock* pBlock); size_t colDataGetNumOfRows(const SSDataBlock* pBlock);
@ -93,13 +90,15 @@ size_t blockDataGetRowSize(const SSDataBlock* pBlock);
double blockDataGetSerialRowSize(const SSDataBlock* pBlock); double blockDataGetSerialRowSize(const SSDataBlock* pBlock);
size_t blockDataGetSerialMetaSize(const SSDataBlock* pBlock); size_t blockDataGetSerialMetaSize(const SSDataBlock* pBlock);
size_t blockDataNumOfRowsForSerialize(const SSDataBlock* pBlock, int32_t blockSize); SSchema* blockDataExtractSchema(const SSDataBlock* pBlock, int32_t* numOfCols);
int32_t blockDataSort(SSDataBlock* pDataBlock, SArray* pOrderInfo, bool nullFirst); int32_t blockDataSort(SSDataBlock* pDataBlock, SArray* pOrderInfo, bool nullFirst);
int32_t blockDataSort_rv(SSDataBlock* pDataBlock, SArray* pOrderInfo, bool nullFirst); int32_t blockDataSort_rv(SSDataBlock* pDataBlock, SArray* pOrderInfo, bool nullFirst);
int32_t blockDataEnsureCapacity(SSDataBlock* pDataBlock, uint32_t numOfRows); int32_t blockDataEnsureCapacity(SSDataBlock* pDataBlock, uint32_t numOfRows);
void blockDataClearup(SSDataBlock* pDataBlock, bool hasVarCol); void blockDataClearup(SSDataBlock* pDataBlock, bool hasVarCol);
SSDataBlock* createOneDataBlock(const SSDataBlock* pDataBlock);
size_t blockDataGetCapacityInRow(const SSDataBlock* pBlock, size_t pageSize);
void *blockDataDestroy(SSDataBlock *pBlock); void *blockDataDestroy(SSDataBlock *pBlock);
#ifdef __cplusplus #ifdef __cplusplus

View File

@ -31,6 +31,7 @@ typedef struct SReadHandle {
void* reader; void* reader;
void* meta; void* meta;
} SReadHandle; } SReadHandle;
/** /**
* Create the exec task for streaming mode * Create the exec task for streaming mode
* @param pMsg * @param pMsg
@ -40,12 +41,22 @@ typedef struct SReadHandle {
qTaskInfo_t qCreateStreamExecTaskInfo(void *msg, void* streamReadHandle); qTaskInfo_t qCreateStreamExecTaskInfo(void *msg, void* streamReadHandle);
/** /**
* * Set the input data block for the stream scan.
* @param tinfo * @param tinfo
* @param input * @param input
* @return * @return
*/ */
int32_t qSetStreamInput(qTaskInfo_t tinfo, const void* input); int32_t qSetStreamInput(qTaskInfo_t tinfo, const void* input);
/**
* Update the table id list, add or remove.
*
* @param tinfo
* @param id
* @param isAdd
* @return
*/
int32_t qUpdateQualifiedTableId(qTaskInfo_t tinfo, SArray* tableIdList, bool isAdd);
/** /**
* Create the exec task object according to task json * Create the exec task object according to task json

View File

@ -138,8 +138,10 @@ extern SFunctionFpSet fpSet[1];
// sql function runtime context // sql function runtime context
typedef struct SqlFunctionCtx { typedef struct SqlFunctionCtx {
int32_t startRow;
int32_t size; // number of rows int32_t size; // number of rows
void * pInput; // input data buffer SColumnInfoData* pInput;
uint32_t order; // asc|desc uint32_t order; // asc|desc
int16_t inputType; int16_t inputType;
int16_t inputBytes; int16_t inputBytes;

View File

@ -47,5 +47,6 @@ OP_ENUM_MACRO(AllTimeWindow)
OP_ENUM_MACRO(AllMultiTableTimeInterval) OP_ENUM_MACRO(AllMultiTableTimeInterval)
OP_ENUM_MACRO(Order) OP_ENUM_MACRO(Order)
OP_ENUM_MACRO(Exchange) OP_ENUM_MACRO(Exchange)
OP_ENUM_MACRO(SortedMerge)
//OP_ENUM_MACRO(TableScan) //OP_ENUM_MACRO(TableScan)

View File

@ -42,8 +42,8 @@ extern "C" {
typedef struct SArray { typedef struct SArray {
size_t size; size_t size;
size_t capacity; uint32_t capacity;
size_t elemSize; uint32_t elemSize;
void* pData; void* pData;
} SArray; } SArray;

View File

@ -29,10 +29,9 @@ typedef struct SPageInfo SPageInfo;
typedef struct SDiskbasedBuf SDiskbasedBuf; typedef struct SDiskbasedBuf SDiskbasedBuf;
#define DEFAULT_INTERN_BUF_PAGE_SIZE (1024L) // in bytes #define DEFAULT_INTERN_BUF_PAGE_SIZE (1024L) // in bytes
#define DEFAULT_PAGE_SIZE (16384L)
typedef struct SFilePage { typedef struct SFilePage {
int64_t num; int32_t num;
char data[]; char data[];
} SFilePage; } SFilePage;
@ -54,8 +53,7 @@ typedef struct SDiskbasedBufStatis {
* @param handle * @param handle
* @return * @return
*/ */
int32_t createDiskbasedBuffer(SDiskbasedBuf** pBuf, int32_t pagesize, int32_t inMemBufSize, uint64_t qId, int32_t createDiskbasedBuf(SDiskbasedBuf** pBuf, int32_t pagesize, int32_t inMemBufSize, uint64_t qId, const char* dir);
const char* dir);
/** /**
* *
@ -64,7 +62,7 @@ int32_t createDiskbasedBuffer(SDiskbasedBuf** pBuf, int32_t pagesize, int32_t in
* @param pageId * @param pageId
* @return * @return
*/ */
SFilePage* getNewDataBuf(SDiskbasedBuf* pBuf, int32_t groupId, int32_t* pageId); void* getNewBufPage(SDiskbasedBuf* pBuf, int32_t groupId, int32_t* pageId);
/** /**
* *
@ -80,7 +78,7 @@ SIDList getDataBufPagesIdList(SDiskbasedBuf* pBuf, int32_t groupId);
* @param id * @param id
* @return * @return
*/ */
SFilePage* getBufPage(SDiskbasedBuf* pBuf, int32_t id); void* getBufPage(SDiskbasedBuf* pBuf, int32_t id);
/** /**
* release the referenced buf pages * release the referenced buf pages
@ -108,13 +106,13 @@ size_t getTotalBufSize(const SDiskbasedBuf* pBuf);
* @param pBuf * @param pBuf
* @return * @return
*/ */
size_t getNumOfResultBufGroupId(const SDiskbasedBuf* pBuf); size_t getNumOfBufGroupId(const SDiskbasedBuf* pBuf);
/** /**
* destroy result buffer * destroy result buffer
* @param pBuf * @param pBuf
*/ */
void destroyResultBuf(SDiskbasedBuf* pBuf); void destroyDiskbasedBuf(SDiskbasedBuf* pBuf);
/** /**
* *
@ -137,6 +135,11 @@ int32_t getPageId(const SPageInfo* pPgInfo);
*/ */
int32_t getBufPageSize(const SDiskbasedBuf* pBuf); int32_t getBufPageSize(const SDiskbasedBuf* pBuf);
/**
*
* @param pBuf
* @return
*/
int32_t getNumOfInMemBufPages(const SDiskbasedBuf* pBuf); int32_t getNumOfInMemBufPages(const SDiskbasedBuf* pBuf);
/** /**
@ -148,22 +151,43 @@ bool isAllDataInMemBuf(const SDiskbasedBuf* pBuf);
/** /**
* Set the buffer page is dirty, and needs to be flushed to disk when swap out. * Set the buffer page is dirty, and needs to be flushed to disk when swap out.
* @param pPageInfo * @param pPage
* @param dirty * @param dirty
*/ */
void setBufPageDirty(SFilePage* pPageInfo, bool dirty); void setBufPageDirty(void* pPage, bool dirty);
/**
* Set the compress/ no-compress flag for paged buffer, when flushing data in disk.
* @param pBuf
*/
void setBufPageCompressOnDisk(SDiskbasedBuf* pBuf, bool comp);
/**
* Set the pageId page buffer is not need
* @param pBuf
* @param pageId
*/
void dBufSetBufPageRecycled(SDiskbasedBuf *pBuf, void* pPage);
/** /**
* Print the statistics when closing this buffer * Print the statistics when closing this buffer
* @param pBuf * @param pBuf
*/ */
void printStatisBeforeClose(SDiskbasedBuf* pBuf); void dBufSetPrintInfo(SDiskbasedBuf* pBuf);
/** /**
* return buf statistics. * Return buf statistics.
* @param pBuf
* @return
*/ */
SDiskbasedBufStatis getDBufStatis(const SDiskbasedBuf* pBuf); SDiskbasedBufStatis getDBufStatis(const SDiskbasedBuf* pBuf);
/**
* Print the buffer statistics information
* @param pBuf
*/
void dBufPrintStatis(const SDiskbasedBuf* pBuf);
#ifdef __cplusplus #ifdef __cplusplus
} }
#endif #endif

View File

@ -53,8 +53,10 @@ TEST(testCase, driverInit_Test) {
// taos_init(); // taos_init();
} }
#if 0 #if 1
TEST(testCase, connect_Test) { TEST(testCase, connect_Test) {
// taos_options(TSDB_OPTION_CONFIGDIR, "/home/ubuntu/first/cfg");
TAOS* pConn = taos_connect("localhost", "root", "taosdata", NULL, 0); TAOS* pConn = taos_connect("localhost", "root", "taosdata", NULL, 0);
if (pConn == NULL) { if (pConn == NULL) {
printf("failed to connect to server, reason:%s\n", taos_errstr(NULL)); printf("failed to connect to server, reason:%s\n", taos_errstr(NULL));

View File

@ -63,7 +63,7 @@ SEpSet getEpSet_s(SCorEpSet *pEpSet) {
#define BitmapLen(_n) (((_n) + ((1<<NBIT)-1)) >> NBIT) #define BitmapLen(_n) (((_n) + ((1<<NBIT)-1)) >> NBIT)
int32_t colDataGetSize(const SColumnInfoData* pColumnInfoData, int32_t numOfRows) { int32_t colDataGetLength(const SColumnInfoData* pColumnInfoData, int32_t numOfRows) {
ASSERT(pColumnInfoData != NULL); ASSERT(pColumnInfoData != NULL);
if (IS_VAR_DATA_TYPE(pColumnInfoData->info.type)) { if (IS_VAR_DATA_TYPE(pColumnInfoData->info.type)) {
return pColumnInfoData->varmeta.length; return pColumnInfoData->varmeta.length;
@ -127,6 +127,7 @@ int32_t colDataAppend(SColumnInfoData* pColumnInfoData, uint32_t currentRow, con
case TSDB_DATA_TYPE_USMALLINT: {*(int16_t*) p = *(int16_t*) pData;break;} case TSDB_DATA_TYPE_USMALLINT: {*(int16_t*) p = *(int16_t*) pData;break;}
case TSDB_DATA_TYPE_INT: case TSDB_DATA_TYPE_INT:
case TSDB_DATA_TYPE_UINT: {*(int32_t*) p = *(int32_t*) pData;break;} case TSDB_DATA_TYPE_UINT: {*(int32_t*) p = *(int32_t*) pData;break;}
case TSDB_DATA_TYPE_TIMESTAMP:
case TSDB_DATA_TYPE_BIGINT: case TSDB_DATA_TYPE_BIGINT:
case TSDB_DATA_TYPE_UBIGINT: {*(int64_t*) p = *(int64_t*) pData;break;} case TSDB_DATA_TYPE_UBIGINT: {*(int64_t*) p = *(int64_t*) pData;break;}
default: default:
@ -249,8 +250,8 @@ int32_t blockDataUpdateTsWindow(SSDataBlock* pDataBlock) {
} }
ASSERT(pColInfoData->nullbitmap == NULL); ASSERT(pColInfoData->nullbitmap == NULL);
pDataBlock->info.window.skey = *(TSKEY*) colDataGet(pColInfoData, 0); pDataBlock->info.window.skey = *(TSKEY*) colDataGetData(pColInfoData, 0);
pDataBlock->info.window.ekey = *(TSKEY*) colDataGet(pColInfoData, (pDataBlock->info.rows - 1)); pDataBlock->info.window.ekey = *(TSKEY*) colDataGetData(pColInfoData, (pDataBlock->info.rows - 1));
return 0; return 0;
} }
@ -262,8 +263,8 @@ int32_t blockDataMerge(SSDataBlock* pDest, const SSDataBlock* pSrc) {
SColumnInfoData* pCol2 = taosArrayGet(pDest->pDataBlock, i); SColumnInfoData* pCol2 = taosArrayGet(pDest->pDataBlock, i);
SColumnInfoData* pCol1 = taosArrayGet(pSrc->pDataBlock, i); SColumnInfoData* pCol1 = taosArrayGet(pSrc->pDataBlock, i);
uint32_t oldLen = colDataGetSize(pCol2, pDest->info.rows); uint32_t oldLen = colDataGetLength(pCol2, pDest->info.rows);
uint32_t newLen = colDataGetSize(pCol1, pSrc->info.rows); uint32_t newLen = colDataGetLength(pCol1, pSrc->info.rows);
int32_t newSize = oldLen + newLen; int32_t newSize = oldLen + newLen;
char* tmp = realloc(pCol2->pData, newSize); char* tmp = realloc(pCol2->pData, newSize);
@ -287,7 +288,7 @@ size_t blockDataGetSize(const SSDataBlock* pBlock) {
for(int32_t i = 0; i < numOfCols; ++i) { for(int32_t i = 0; i < numOfCols; ++i) {
SColumnInfoData* pColInfoData = taosArrayGet(pBlock->pDataBlock, i); SColumnInfoData* pColInfoData = taosArrayGet(pBlock->pDataBlock, i);
total += colDataGetSize(pColInfoData, pBlock->info.rows); total += colDataGetLength(pColInfoData, pBlock->info.rows);
if (IS_VAR_DATA_TYPE(pColInfoData->info.type)) { if (IS_VAR_DATA_TYPE(pColInfoData->info.type)) {
total += sizeof(int32_t) * pBlock->info.rows; total += sizeof(int32_t) * pBlock->info.rows;
@ -336,7 +337,7 @@ int32_t blockDataSplitRows(SSDataBlock* pBlock, bool hasVarCol, int32_t startInd
if (isNull) { if (isNull) {
// do nothing // do nothing
} else { } else {
char* p = colDataGet(pColInfoData, j); char* p = colDataGetData(pColInfoData, j);
size += varDataTLen(p); size += varDataTLen(p);
} }
@ -401,7 +402,7 @@ SSDataBlock* blockDataExtractBlock(SSDataBlock* pBlock, int32_t startIndex, int3
for (int32_t j = startIndex; j < (startIndex + rowCount); ++j) { for (int32_t j = startIndex; j < (startIndex + rowCount); ++j) {
bool isNull = colDataIsNull(pColData, pBlock->info.rows, j, pBlock->pBlockAgg); bool isNull = colDataIsNull(pColData, pBlock->info.rows, j, pBlock->pBlockAgg);
char* p = colDataGet(pColData, j); char* p = colDataGetData(pColData, j);
colDataAppend(pDstCol, j - startIndex, p, isNull); colDataAppend(pDstCol, j - startIndex, p, isNull);
} }
@ -411,7 +412,6 @@ SSDataBlock* blockDataExtractBlock(SSDataBlock* pBlock, int32_t startIndex, int3
return pDst; return pDst;
} }
/** /**
* *
* +------------------+---------------+--------------------+ * +------------------+---------------+--------------------+
@ -444,7 +444,7 @@ int32_t blockDataToBuf(char* buf, const SSDataBlock* pBlock) {
pStart += BitmapLen(pBlock->info.rows); pStart += BitmapLen(pBlock->info.rows);
} }
uint32_t dataSize = colDataGetSize(pCol, numOfRows); uint32_t dataSize = colDataGetLength(pCol, numOfRows);
*(int32_t*) pStart = dataSize; *(int32_t*) pStart = dataSize;
pStart += sizeof(int32_t); pStart += sizeof(int32_t);
@ -522,6 +522,22 @@ size_t blockDataGetSerialMetaSize(const SSDataBlock* pBlock) {
return sizeof(int32_t) + pBlock->info.numOfCols * sizeof(int32_t); return sizeof(int32_t) + pBlock->info.numOfCols * sizeof(int32_t);
} }
SSchema* blockDataExtractSchema(const SSDataBlock* pBlock, int32_t* numOfCols) {
SSchema* pSchema = calloc(pBlock->info.numOfCols, sizeof(SSchema));
for(int32_t i = 0; i < pBlock->info.numOfCols; ++i) {
SColumnInfoData* pColInfoData = taosArrayGet(pBlock->pDataBlock, i);
pSchema[i].bytes = pColInfoData->info.bytes;
pSchema[i].type = pColInfoData->info.type;
pSchema[i].colId = pColInfoData->info.colId;
}
if (numOfCols != NULL) {
*numOfCols = pBlock->info.numOfCols;
}
return pSchema;
}
double blockDataGetSerialRowSize(const SSDataBlock* pBlock) { double blockDataGetSerialRowSize(const SSDataBlock* pBlock) {
ASSERT(pBlock != NULL); ASSERT(pBlock != NULL);
double rowSize = 0; double rowSize = 0;
@ -577,8 +593,8 @@ int32_t dataBlockCompar(const void* p1, const void* p2, const void* param) {
} }
} }
void* left1 = colDataGet(pColInfoData, left); void* left1 = colDataGetData(pColInfoData, left);
void* right1 = colDataGet(pColInfoData, right); void* right1 = colDataGetData(pColInfoData, right);
switch(pColInfoData->info.type) { switch(pColInfoData->info.type) {
case TSDB_DATA_TYPE_INT: { case TSDB_DATA_TYPE_INT: {
@ -617,7 +633,7 @@ static int32_t doAssignOneTuple(SColumnInfoData* pDstCols, int32_t numOfRows, co
return code; return code;
} }
} else { } else {
char* p = colDataGet(pSrc, tupleIndex); char* p = colDataGetData(pSrc, tupleIndex);
code = colDataAppend(pDst, numOfRows, p, false); code = colDataAppend(pDst, numOfRows, p, false);
if (code != TSDB_CODE_SUCCESS) { if (code != TSDB_CODE_SUCCESS) {
return code; return code;
@ -956,8 +972,8 @@ int32_t dataBlockCompar_rv(const void* p1, const void* p2, const void* param) {
// } // }
// } // }
// void* left1 = colDataGet(pColInfoData, left); // void* left1 = colDataGetData(pColInfoData, left);
// void* right1 = colDataGet(pColInfoData, right); // void* right1 = colDataGetData(pColInfoData, right);
// switch(pColInfoData->info.type) { // switch(pColInfoData->info.type) {
// case TSDB_DATA_TYPE_INT: { // case TSDB_DATA_TYPE_INT: {
@ -1098,4 +1114,25 @@ void* blockDataDestroy(SSDataBlock* pBlock) {
tfree(pBlock->pBlockAgg); tfree(pBlock->pBlockAgg);
tfree(pBlock); tfree(pBlock);
return NULL; return NULL;
}
SSDataBlock* createOneDataBlock(const SSDataBlock* pDataBlock) {
int32_t numOfCols = pDataBlock->info.numOfCols;
SSDataBlock* pBlock = calloc(1, sizeof(SSDataBlock));
pBlock->pDataBlock = taosArrayInit(numOfCols, sizeof(SColumnInfoData));
pBlock->info.numOfCols = numOfCols;
for(int32_t i = 0; i < numOfCols; ++i) {
SColumnInfoData colInfo = {0};
SColumnInfoData* p = taosArrayGet(pDataBlock->pDataBlock, i);
colInfo.info = p->info;
taosArrayPush(pBlock->pDataBlock, &colInfo);
}
return pBlock;
}
size_t blockDataGetCapacityInRow(const SSDataBlock* pBlock, size_t pageSize) {
return pageSize / (blockDataGetSerialRowSize(pBlock) + blockDataGetSerialMetaSize(pBlock));
} }

View File

@ -162,7 +162,7 @@ TEST(testCase, Datablock_test) {
ASSERT_EQ(colDataGetNumOfCols(b), 2); ASSERT_EQ(colDataGetNumOfCols(b), 2);
ASSERT_EQ(colDataGetNumOfRows(b), 40); ASSERT_EQ(colDataGetNumOfRows(b), 40);
char* pData = colDataGet(p1, 3); char* pData = colDataGetData(p1, 3);
printf("the second row of binary:%s, length:%d\n", (char*)varDataVal(pData), varDataLen(pData)); printf("the second row of binary:%s, length:%d\n", (char*)varDataVal(pData), varDataLen(pData));
SArray* pOrderInfo = taosArrayInit(3, sizeof(SBlockOrderInfo)); SArray* pOrderInfo = taosArrayInit(3, sizeof(SBlockOrderInfo));

View File

@ -216,13 +216,15 @@ static FORCE_INLINE void tqReadHandleSetColIdList(STqReadHandle *pReadHandle, SA
static FORCE_INLINE int tqReadHandleSetTbUidList(STqReadHandle *pHandle, const SArray *tbUidList) { static FORCE_INLINE int tqReadHandleSetTbUidList(STqReadHandle *pHandle, const SArray *tbUidList) {
pHandle->tbIdHash = taosHashInit(64, taosGetDefaultHashFunction(TSDB_DATA_TYPE_BIGINT), true, HASH_NO_LOCK); pHandle->tbIdHash = taosHashInit(64, taosGetDefaultHashFunction(TSDB_DATA_TYPE_BIGINT), true, HASH_NO_LOCK);
if (pHandle->tbIdHash == NULL) { if (pHandle->tbIdHash == NULL) {
terrno = TSDB_CODE_OUT_OF_MEMORY;
return -1; return -1;
} }
for (int i = 0; i < taosArrayGetSize(tbUidList); i++) { for (int i = 0; i < taosArrayGetSize(tbUidList); i++) {
int64_t *pKey = (int64_t *)taosArrayGet(tbUidList, i); int64_t *pKey = (int64_t *)taosArrayGet(tbUidList, i);
taosHashPut(pHandle->tbIdHash, pKey, sizeof(int64_t), NULL, 0); taosHashPut(pHandle->tbIdHash, pKey, sizeof(int64_t), NULL, 0);
// pHandle->tbUid = tbUid;
} }
return 0; return 0;
} }

View File

@ -88,7 +88,7 @@ typedef struct STableCheckInfo {
int32_t compSize; int32_t compSize;
int32_t numOfBlocks:29; // number of qualified data blocks not the original blocks int32_t numOfBlocks:29; // number of qualified data blocks not the original blocks
uint8_t chosen:2; // indicate which iterator should move forward uint8_t chosen:2; // indicate which iterator should move forward
bool initBuf; // whether to initialize the in-memory skip list iterator or not bool initBuf:1; // whether to initialize the in-memory skip list iterator or not
SSkipListIterator* iter; // mem buffer skip list iterator SSkipListIterator* iter; // mem buffer skip list iterator
SSkipListIterator* iiter; // imem buffer skip list iterator SSkipListIterator* iiter; // imem buffer skip list iterator
} STableCheckInfo; } STableCheckInfo;

View File

@ -68,9 +68,10 @@ typedef struct SResultRow {
} SResultRow; } SResultRow;
typedef struct SResultRowInfo { typedef struct SResultRowInfo {
SResultRow *pCurResult; // current active result row info
SResultRow** pResult; // result list SResultRow** pResult; // result list
int16_t type:8; // data type for hash key // int16_t type:8; // data type for hash key
int32_t size:24; // number of result set int32_t size; // number of result set
int32_t capacity; // max capacity int32_t capacity; // max capacity
int32_t curPos; // current active result row index of pResult list int32_t curPos; // current active result row index of pResult list
} SResultRowInfo; } SResultRowInfo;
@ -95,7 +96,7 @@ struct SUdfInfo;
int32_t getOutputInterResultBufSize(struct STaskAttr* pQueryAttr); int32_t getOutputInterResultBufSize(struct STaskAttr* pQueryAttr);
size_t getResultRowSize(SArray* pExprInfo); size_t getResultRowSize(SArray* pExprInfo);
int32_t initResultRowInfo(SResultRowInfo* pResultRowInfo, int32_t size, int16_t type); int32_t initResultRowInfo(SResultRowInfo* pResultRowInfo, int32_t size);
void cleanupResultRowInfo(SResultRowInfo* pResultRowInfo); void cleanupResultRowInfo(SResultRowInfo* pResultRowInfo);
void resetResultRowInfo(struct STaskRuntimeEnv* pRuntimeEnv, SResultRowInfo* pResultRowInfo); void resetResultRowInfo(struct STaskRuntimeEnv* pRuntimeEnv, SResultRowInfo* pResultRowInfo);
@ -105,7 +106,7 @@ void closeAllResultRows(SResultRowInfo* pResultRowInfo);
int32_t initResultRow(SResultRow *pResultRow); int32_t initResultRow(SResultRow *pResultRow);
void closeResultRow(SResultRowInfo* pResultRowInfo, int32_t slot); void closeResultRow(SResultRowInfo* pResultRowInfo, int32_t slot);
bool isResultRowClosed(SResultRowInfo *pResultRowInfo, int32_t slot); bool isResultRowClosed(SResultRowInfo *pResultRowInfo, int32_t slot);
void clearResultRow(struct STaskRuntimeEnv* pRuntimeEnv, SResultRow* pResultRow, int16_t type); void clearResultRow(struct STaskRuntimeEnv* pRuntimeEnv, SResultRow* pResultRow);
struct SResultRowEntryInfo* getResultCell(const SResultRow* pRow, int32_t index, int32_t* offset); struct SResultRowEntryInfo* getResultCell(const SResultRow* pRow, int32_t index, int32_t* offset);

View File

@ -15,6 +15,7 @@
#ifndef TDENGINE_EXECUTORIMPL_H #ifndef TDENGINE_EXECUTORIMPL_H
#define TDENGINE_EXECUTORIMPL_H #define TDENGINE_EXECUTORIMPL_H
#include "tsort.h"
#ifdef __cplusplus #ifdef __cplusplus
extern "C" { extern "C" {
#endif #endif
@ -444,16 +445,32 @@ typedef struct SOptrBasicInfo {
int32_t capacity; int32_t capacity;
} SOptrBasicInfo; } SOptrBasicInfo;
typedef struct SOptrBasicInfo STableIntervalOperatorInfo; typedef struct SAggSupporter {
typedef struct SAggOperatorInfo {
SOptrBasicInfo binfo;
SDiskbasedBuf *pResultBuf; // query result buffer based on blocked-wised disk file
SHashObj* pResultRowHashTable; // quick locate the window object for each result SHashObj* pResultRowHashTable; // quick locate the window object for each result
SHashObj* pResultRowListSet; // used to check if current ResultRowInfo has ResultRow object or not SHashObj* pResultRowListSet; // used to check if current ResultRowInfo has ResultRow object or not
SArray* pResultRowArrayList; // The array list that contains the Result rows SArray* pResultRowArrayList; // The array list that contains the Result rows
char* keyBuf; // window key buffer char* keyBuf; // window key buffer
SResultRowPool *pool; // The window result objects pool, all the resultRow Objects are allocated and managed by this object. SResultRowPool *pool; // The window result objects pool, all the resultRow Objects are allocated and managed by this object.
} SAggSupporter;
typedef struct STableIntervalOperatorInfo {
SOptrBasicInfo binfo;
SDiskbasedBuf *pResultBuf; // query result buffer based on blocked-wised disk file
SGroupResInfo groupResInfo;
SInterval interval;
STimeWindow win;
int32_t precision;
bool timeWindowInterpo;
char **pRow;
SAggSupporter aggSup;
STableQueryInfo *pCurrent;
int32_t order;
} STableIntervalOperatorInfo;
typedef struct SAggOperatorInfo {
SOptrBasicInfo binfo;
SDiskbasedBuf *pResultBuf; // query result buffer based on blocked-wised disk file
SAggSupporter aggSup;
STableQueryInfo *current; STableQueryInfo *current;
uint32_t groupId; uint32_t groupId;
SGroupResInfo groupResInfo; SGroupResInfo groupResInfo;
@ -549,49 +566,42 @@ typedef struct SDistinctOperatorInfo {
SArray* pDistinctDataInfo; SArray* pDistinctDataInfo;
} SDistinctOperatorInfo; } SDistinctOperatorInfo;
struct SGlobalMerger; typedef struct SSortedMergeOperatorInfo {
SOptrBasicInfo binfo;
bool hasVarCol;
SArray *orderInfo; // SArray<SBlockOrderInfo>
bool nullFirst;
int32_t numOfSources;
typedef struct SMultiwayMergeInfo { SSortHandle *pSortHandle;
struct SGlobalMerger* pMerge;
SOptrBasicInfo binfo;
int32_t bufCapacity;
int64_t seed;
char** prevRow;
SArray* orderColumnList;
int32_t resultRowFactor;
bool hasGroupColData; int32_t bufPageSize;
char** currentGroupColData; uint32_t sortBufSize; // max buffer size for in-memory sort
SArray* groupColumnList;
bool hasDataBlockForNewGroup;
SSDataBlock* pExistBlock;
SArray* udfInfo; int32_t resultRowFactor;
bool hasPrev; bool hasGroupVal;
bool multiGroupResults;
} SMultiwayMergeInfo;
typedef struct SMsortComparParam { SDiskbasedBuf *pTupleStore; // keep the final results
struct SExternalMemSource **pSources; int32_t numOfResPerPage;
int32_t numOfSources;
SArray *orderInfo; // SArray<SBlockOrderInfo> char** groupVal;
bool nullFirst; SArray *groupInfo;
} SMsortComparParam; SAggSupporter aggSup;
} SSortedMergeOperatorInfo;
typedef struct SOrderOperatorInfo { typedef struct SOrderOperatorInfo {
int32_t sourceId;
uint32_t sortBufSize; // max buffer size for in-memory sort uint32_t sortBufSize; // max buffer size for in-memory sort
SSDataBlock *pDataBlock; SSDataBlock *pDataBlock;
bool hasVarCol; // has variable length column, such as binary/varchar/nchar bool hasVarCol; // has variable length column, such as binary/varchar/nchar
int32_t numOfCompleted; SArray *orderInfo;
SDiskbasedBuf *pSortInternalBuf; bool nullFirst;
SMultiwayMergeTreeInfo *pMergeTree; SSortHandle *pSortHandle;
SArray *pSources; // SArray<SExternalMemSource*>
int32_t bufPageSize; int32_t bufPageSize;
int32_t numOfRowsInRes; int32_t numOfRowsInRes;
SMsortComparParam cmpParam; // TODO extact struct
int64_t startTs; // sort start time int64_t startTs; // sort start time
uint64_t sortElapsed; // sort elapsed time, time to flush to disk not included. uint64_t sortElapsed; // sort elapsed time, time to flush to disk not included.
uint64_t totalSize; // total load bytes from remote uint64_t totalSize; // total load bytes from remote
@ -608,8 +618,8 @@ SOperatorInfo* createMultiTableAggOperatorInfo(SOperatorInfo* downstream, SArray
SOperatorInfo* createProjectOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream, SExprInfo* pExpr, SOperatorInfo* createProjectOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream, SExprInfo* pExpr,
int32_t numOfOutput); int32_t numOfOutput);
SOperatorInfo* createLimitOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream); SOperatorInfo* createLimitOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream);
SOperatorInfo* createTimeIntervalOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream, SExprInfo* pExpr, SOperatorInfo* createIntervalOperatorInfo(SOperatorInfo* downstream, SArray* pExprInfo, SExecTaskInfo* pTaskInfo);
int32_t numOfOutput);
SOperatorInfo* createAllTimeIntervalOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream, SOperatorInfo* createAllTimeIntervalOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream,
SExprInfo* pExpr, int32_t numOfOutput); SExprInfo* pExpr, int32_t numOfOutput);
SOperatorInfo* createSWindowOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream, SExprInfo* pExpr, SOperatorInfo* createSWindowOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorInfo* downstream, SExprInfo* pExpr,
@ -641,9 +651,8 @@ SOperatorInfo* createFilterOperatorInfo(STaskRuntimeEnv* pRuntimeEnv, SOperatorI
SOperatorInfo* createJoinOperatorInfo(SOperatorInfo** pdownstream, int32_t numOfDownstream, SSchema* pSchema, SOperatorInfo* createJoinOperatorInfo(SOperatorInfo** pdownstream, int32_t numOfDownstream, SSchema* pSchema,
int32_t numOfOutput); int32_t numOfOutput);
SOperatorInfo* createOrderOperatorInfo(SOperatorInfo* downstream, SArray* pExprInfo, SArray* pOrderVal); SOperatorInfo* createOrderOperatorInfo(SOperatorInfo* downstream, SArray* pExprInfo, SArray* pOrderVal, SExecTaskInfo* pTaskInfo);
SOperatorInfo* createMergeSortOperatorInfo(SOperatorInfo* downstream, SExprInfo* pExpr, int32_t numOfOutput, SOperatorInfo* createSortedMergeOperatorInfo(SOperatorInfo** downstream, int32_t numOfDownstream, SArray* pExprInfo, SArray* pOrderVal, SArray* pGroupInfo, SExecTaskInfo* pTaskInfo);
SOrder* pOrderVal);
// SSDataBlock* doGlobalAggregate(void* param, bool* newgroup); // SSDataBlock* doGlobalAggregate(void* param, bool* newgroup);
// SSDataBlock* doMultiwayMergeSort(void* param, bool* newgroup); // SSDataBlock* doMultiwayMergeSort(void* param, bool* newgroup);
@ -691,9 +700,6 @@ int32_t checkForQueryBuf(size_t numOfTables);
bool checkNeedToCompressQueryCol(SQInfo* pQInfo); bool checkNeedToCompressQueryCol(SQInfo* pQInfo);
void setQueryStatus(STaskRuntimeEnv* pRuntimeEnv, int8_t status); void setQueryStatus(STaskRuntimeEnv* pRuntimeEnv, int8_t status);
bool onlyQueryTags(STaskAttr* pQueryAttr);
// void destroyUdfInfo(struct SUdfInfo* pUdfInfo);
int32_t doDumpQueryResult(SQInfo* pQInfo, char* data, int8_t compressed, int32_t* compLen); int32_t doDumpQueryResult(SQInfo* pQInfo, char* data, int8_t compressed, int32_t* compLen);
size_t getResultSize(SQInfo* pQInfo, int64_t* numOfRows); size_t getResultSize(SQInfo* pQInfo, int64_t* numOfRows);

View File

@ -0,0 +1,44 @@
/*
* Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
*
* This program is free software: you can use, redistribute, and/or modify
* it under the terms of the GNU Affero General Public License, version 3
* or later ("AGPL"), as published by the Free Software Foundation.
*
* This program is distributed in the hope that it will be useful, but WITHOUT
* ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
* FITNESS FOR A PARTICULAR PURPOSE.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
#ifndef TDENGINE_TLINEARHASH_H
#define TDENGINE_TLINEARHASH_H
#ifdef __cplusplus
extern "C" {
#endif
#include "thash.h"
enum {
LINEAR_HASH_STATIS = 0x1,
LINEAR_HASH_DATA = 0x2,
};
typedef struct SLHashObj SLHashObj;
SLHashObj* tHashInit(int32_t inMemPages, int32_t pageSize, _hash_fn_t fn, int32_t numOfTuplePerPage);
void* tHashCleanup(SLHashObj* pHashObj);
int32_t tHashPut(SLHashObj* pHashObj, const void *key, size_t keyLen, void *data, size_t size);
char* tHashGet(SLHashObj* pHashObj, const void *key, size_t keyLen);
int32_t tHashRemove(SLHashObj* pHashObj, const void *key, size_t keyLen);
void tHashPrint(const SLHashObj* pHashObj, int32_t type);
#ifdef __cplusplus
}
#endif
#endif // TDENGINE_TLINEARHASH_H

View File

@ -0,0 +1,137 @@
/*
* Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
*
* This program is free software: you can use, redistribute, and/or modify
* it under the terms of the GNU Affero General Public License, version 3
* or later ("AGPL"), as published by the Free Software Foundation.
*
* This program is distributed in the hope that it will be useful, but WITHOUT
* ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
* FITNESS FOR A PARTICULAR PURPOSE.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
#ifndef TDENGINE_TSORT_H
#define TDENGINE_TSORT_H
#ifdef __cplusplus
extern "C" {
#endif
#include "common.h"
#include "os.h"
enum {
SORT_MULTISOURCE_MERGE = 0x1,
SORT_SINGLESOURCE_SORT = 0x2,
};
typedef struct SMultiMergeSource {
int32_t type;
int32_t rowIndex;
SSDataBlock *pBlock;
} SMultiMergeSource;
typedef struct SExternalMemSource {
SMultiMergeSource src;
SArray* pageIdList;
int32_t pageIndex;
} SExternalMemSource;
typedef struct SGenericSource {
SMultiMergeSource src;
void *param;
} SGenericSource;
typedef struct SMsortComparParam {
void **pSources;
int32_t numOfSources;
SArray *orderInfo; // SArray<SBlockOrderInfo>
bool nullFirst;
} SMsortComparParam;
typedef struct SSortHandle SSortHandle;
typedef struct STupleHandle STupleHandle;
typedef SSDataBlock* (*_sort_fetch_block_fn_t)(void* param);
typedef int32_t (*_sort_merge_compar_fn_t)(const void* p1, const void* p2, void* param);
/**
*
* @param type
* @return
*/
SSortHandle* tsortCreateSortHandle(SArray* pOrderInfo, bool nullFirst, int32_t type, int32_t pageSize, int32_t numOfPages, SSchema* pSchema, int32_t numOfCols, const char* idstr);
/**
*
* @param pSortHandle
*/
void tsortDestroySortHandle(SSortHandle* pSortHandle);
/**
*
* @param pHandle
* @return
*/
int32_t tsortOpen(SSortHandle* pHandle);
/**
*
* @param pHandle
* @return
*/
int32_t tsortClose(SSortHandle* pHandle);
/**
*
* @return
*/
int32_t tsortSetFetchRawDataFp(SSortHandle* pHandle, _sort_fetch_block_fn_t fp);
/**
*
* @param pHandle
* @param fp
* @return
*/
int32_t tsortSetComparFp(SSortHandle* pHandle, _sort_merge_compar_fn_t fp);
/**
*
* @param pHandle
* @param pSource
* @return success or failed
*/
int32_t tsortAddSource(SSortHandle* pSortHandle, void* pSource);
/**
*
* @param pHandle
* @return
*/
STupleHandle* tsortNextTuple(SSortHandle* pHandle);
/**
*
* @param pHandle
* @param colIndex
* @return
*/
bool tsortIsNullVal(STupleHandle* pVHandle, int32_t colIndex);
/**
*
* @param pHandle
* @param colIndex
* @return
*/
void* tsortGetValue(STupleHandle* pVHandle, int32_t colIndex);
#ifdef __cplusplus
}
#endif
#endif // TDENGINE_TSORT_H

View File

@ -53,8 +53,7 @@ int32_t getOutputInterResultBufSize(STaskAttr* pQueryAttr) {
return size; return size;
} }
int32_t initResultRowInfo(SResultRowInfo *pResultRowInfo, int32_t size, int16_t type) { int32_t initResultRowInfo(SResultRowInfo *pResultRowInfo, int32_t size) {
pResultRowInfo->type = type;
pResultRowInfo->size = 0; pResultRowInfo->size = 0;
pResultRowInfo->curPos = -1; pResultRowInfo->curPos = -1;
pResultRowInfo->capacity = size; pResultRowInfo->capacity = size;
@ -93,7 +92,7 @@ void resetResultRowInfo(STaskRuntimeEnv *pRuntimeEnv, SResultRowInfo *pResultRow
for (int32_t i = 0; i < pResultRowInfo->size; ++i) { for (int32_t i = 0; i < pResultRowInfo->size; ++i) {
SResultRow *pWindowRes = pResultRowInfo->pResult[i]; SResultRow *pWindowRes = pResultRowInfo->pResult[i];
clearResultRow(pRuntimeEnv, pWindowRes, pResultRowInfo->type); clearResultRow(pRuntimeEnv, pWindowRes);
int32_t groupIndex = 0; int32_t groupIndex = 0;
int64_t uid = 0; int64_t uid = 0;
@ -136,7 +135,7 @@ void closeResultRow(SResultRowInfo *pResultRowInfo, int32_t slot) {
getResultRow(pResultRowInfo, slot)->closed = true; getResultRow(pResultRowInfo, slot)->closed = true;
} }
void clearResultRow(STaskRuntimeEnv *pRuntimeEnv, SResultRow *pResultRow, int16_t type) { void clearResultRow(STaskRuntimeEnv *pRuntimeEnv, SResultRow *pResultRow) {
if (pResultRow == NULL) { if (pResultRow == NULL) {
return; return;
} }

View File

@ -93,3 +93,25 @@ qTaskInfo_t qCreateStreamExecTaskInfo(void* msg, void* streamReadHandle) {
return pTaskInfo; return pTaskInfo;
} }
int32_t qUpdateQualifiedTableId(qTaskInfo_t tinfo, SArray* tableIdList, bool isAdd) {
SExecTaskInfo* pTaskInfo = (SExecTaskInfo* )tinfo;
// traverse to the streamscan node to add this table id
SOperatorInfo* pInfo = pTaskInfo->pRoot;
while(pInfo->operatorType != OP_StreamScan) {
pInfo = pInfo->pDownstream[0];
}
SStreamBlockScanInfo* pScanInfo = pInfo->info;
if (isAdd) {
int32_t code = tqReadHandleSetTbUidList(pScanInfo->readerHandle, tableIdList);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
} else {
assert(0);
}
return TSDB_CODE_SUCCESS;
}

File diff suppressed because it is too large Load Diff

View File

@ -0,0 +1,427 @@
/*
* Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
*
* This program is free software: you can use, redistribute, and/or modify
* it under the terms of the GNU Affero General Public License, version 3
* or later ("AGPL"), as published by the Free Software Foundation.
*
* This program is distributed in the hope that it will be useful, but WITHOUT
* ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
* FITNESS FOR A PARTICULAR PURPOSE.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
#include "tlinearhash.h"
#include "tcfg.h"
#include "taoserror.h"
#include "tpagedbuf.h"
#define LHASH_CAP_RATIO 0.85
// Always located in memory
typedef struct SLHashBucket {
SArray *pPageIdList;
int32_t size; // the number of element in this entry
} SLHashBucket;
typedef struct SLHashObj {
SDiskbasedBuf *pBuf;
_hash_fn_t hashFn;
int32_t tuplesPerPage;
SLHashBucket **pBucket; // entry list
int32_t numOfAlloc; // number of allocated bucket ptr slot
int32_t bits; // the number of bits used in hash
int32_t numOfBuckets; // the number of buckets
int64_t size; // the number of total items
} SLHashObj;
/**
* the data struct for each hash node
* +-----------+-------+--------+
* | SLHashNode| key | data |
* +-----------+-------+--------+
*/
typedef struct SLHashNode {
uint16_t keyLen;
uint16_t dataLen;
} SLHashNode;
#define GET_LHASH_NODE_KEY(_n) (((char*)(_n)) + sizeof(SLHashNode))
#define GET_LHASH_NODE_DATA(_n) ((char*)(_n) + sizeof(SLHashNode) + ((SLHashNode*)(_n))->keyLen)
#define GET_LHASH_NODE_LEN(_n) (sizeof(SLHashNode) + ((SLHashNode*)(_n))->keyLen + ((SLHashNode*)(_n))->dataLen)
static int32_t doAddNewBucket(SLHashObj* pHashObj);
static int32_t doGetBucketIdFromHashVal(int32_t hashv, int32_t bits) {
return hashv & ((1ul << (bits)) - 1);
}
static int32_t doGetAlternativeBucketId(int32_t bucketId, int32_t bits, int32_t numOfBuckets) {
int32_t v = bucketId - (1ul << (bits - 1));
ASSERT(v < numOfBuckets);
return v;
}
static int32_t doGetRelatedSplitBucketId(int32_t bucketId, int32_t bits) {
int32_t splitBucketId = (1ul << (bits - 1)) ^ bucketId;
return splitBucketId;
}
static void doCopyObject(char* p, const void* key, int32_t keyLen, const void* data, int32_t size) {
*(uint16_t*) p = keyLen;
p += sizeof(uint16_t);
*(uint16_t*) p = size;
p += sizeof(uint16_t);
memcpy(p, key, keyLen);
p += keyLen;
memcpy(p, data, size);
}
static int32_t doAddToBucket(SLHashObj* pHashObj, SLHashBucket* pBucket, int32_t index, const void* key, int32_t keyLen,
const void* data, int32_t size) {
int32_t pageId = *(int32_t*)taosArrayGetLast(pBucket->pPageIdList);
SFilePage* pPage = getBufPage(pHashObj->pBuf, pageId);
ASSERT (pPage != NULL);
// put to current buf page
size_t nodeSize = sizeof(SLHashNode) + keyLen + size;
ASSERT(nodeSize + sizeof(SFilePage) <= getBufPageSize(pHashObj->pBuf));
if (pPage->num + nodeSize > getBufPageSize(pHashObj->pBuf)) {
releaseBufPage(pHashObj->pBuf, pPage);
// allocate the overflow buffer page to hold this k/v.
int32_t newPageId = -1;
SFilePage* pNewPage = getNewBufPage(pHashObj->pBuf, 0, &newPageId);
if (pNewPage == NULL) {
return TSDB_CODE_OUT_OF_MEMORY;
}
taosArrayPush(pBucket->pPageIdList, &newPageId);
doCopyObject(pNewPage->data, key, keyLen, data, size);
pNewPage->num = sizeof(SFilePage) + nodeSize;
setBufPageDirty(pNewPage, true);
releaseBufPage(pHashObj->pBuf, pNewPage);
} else {
char* p = (char*) pPage + pPage->num;
doCopyObject(p, key, keyLen, data, size);
pPage->num += nodeSize;
setBufPageDirty(pPage, true);
releaseBufPage(pHashObj->pBuf, pPage);
}
pBucket->size += 1;
// printf("===> add to bucket:0x%x, num:%d, key:%d\n", index, pBucket->size, *(int*) key);
return TSDB_CODE_SUCCESS;
}
static void doRemoveFromBucket(SFilePage* pPage, SLHashNode* pNode, SLHashBucket* pBucket) {
ASSERT(pPage != NULL && pNode != NULL && pBucket->size >= 1);
int32_t len = GET_LHASH_NODE_LEN(pNode);
char* p = (char*) pNode + len;
char* pEnd = (char*)pPage + pPage->num;
memmove(pNode, p, (pEnd - p));
pPage->num -= len;
if (pPage->num == 0) {
// this page is empty, could be recycle in the future.
}
setBufPageDirty(pPage, true);
pBucket->size -= 1;
}
static void doCompressBucketPages(SLHashObj *pHashObj, SLHashBucket* pBucket) {
size_t numOfPages = taosArrayGetSize(pBucket->pPageIdList);
if (numOfPages <= 1) {
return;
}
int32_t* firstPage = taosArrayGet(pBucket->pPageIdList, 0);
SFilePage* pFirst = getBufPage(pHashObj->pBuf, *firstPage);
int32_t* pageId = taosArrayGetLast(pBucket->pPageIdList);
SFilePage* pLast = getBufPage(pHashObj->pBuf, *pageId);
if (pLast->num <= sizeof(SFilePage)) {
// this is empty
dBufSetBufPageRecycled(pHashObj->pBuf, pLast);
releaseBufPage(pHashObj->pBuf, pFirst);
taosArrayRemove(pBucket->pPageIdList, numOfPages - 1);
return;
}
char* pStart = pLast->data;
int32_t nodeSize = GET_LHASH_NODE_LEN(pStart);
while (1) {
if (pFirst->num + nodeSize < getBufPageSize(pHashObj->pBuf)) {
char* p = ((char*)pFirst) + pFirst->num;
SLHashNode* pNode = (SLHashNode*)pStart;
doCopyObject(p, GET_LHASH_NODE_KEY(pStart), pNode->keyLen, GET_LHASH_NODE_DATA(pStart), pNode->dataLen);
setBufPageDirty(pFirst, true);
setBufPageDirty(pLast, true);
ASSERT(pLast->num >= nodeSize + sizeof(SFilePage));
pFirst->num += nodeSize;
pLast->num -= nodeSize;
pStart += nodeSize;
if (pLast->num <= sizeof(SFilePage)) {
// this is empty
dBufSetBufPageRecycled(pHashObj->pBuf, pLast);
releaseBufPage(pHashObj->pBuf, pFirst);
taosArrayRemove(pBucket->pPageIdList, numOfPages - 1);
break;
}
nodeSize = GET_LHASH_NODE_LEN(pStart);
} else { // move to the front of pLast page
if (pStart != pLast->data) {
memmove(pLast->data, pStart, (((char*)pLast) + pLast->num - pStart));
setBufPageDirty(pLast, true);
}
releaseBufPage(pHashObj->pBuf, pLast);
releaseBufPage(pHashObj->pBuf, pFirst);
break;
}
}
}
static int32_t doAddNewBucket(SLHashObj* pHashObj) {
if (pHashObj->numOfBuckets + 1 > pHashObj->numOfAlloc) {
int32_t newLen = pHashObj->numOfAlloc * 1.25;
if (newLen == pHashObj->numOfAlloc) {
newLen += 4;
}
char* p = realloc(pHashObj->pBucket, POINTER_BYTES * newLen);
if (p == NULL) {
return TSDB_CODE_OUT_OF_MEMORY;
}
memset(p + POINTER_BYTES * pHashObj->numOfBuckets, 0, newLen - pHashObj->numOfBuckets);
pHashObj->pBucket = (SLHashBucket**) p;
pHashObj->numOfAlloc = newLen;
}
SLHashBucket* pBucket = calloc(1, sizeof(SLHashBucket));
pHashObj->pBucket[pHashObj->numOfBuckets] = pBucket;
pBucket->pPageIdList = taosArrayInit(2, sizeof(int32_t));
if (pBucket->pPageIdList == NULL || pBucket == NULL) {
return TSDB_CODE_OUT_OF_MEMORY;
}
int32_t pageId = -1;
SFilePage* p = getNewBufPage(pHashObj->pBuf, 0, &pageId);
p->num = sizeof(SFilePage);
setBufPageDirty(p, true);
releaseBufPage(pHashObj->pBuf, p);
taosArrayPush(pBucket->pPageIdList, &pageId);
pHashObj->numOfBuckets += 1;
// printf("---------------add new bucket, id:0x%x, total:%d\n", pHashObj->numOfBuckets - 1, pHashObj->numOfBuckets);
return TSDB_CODE_SUCCESS;
}
SLHashObj* tHashInit(int32_t inMemPages, int32_t pageSize, _hash_fn_t fn, int32_t numOfTuplePerPage) {
SLHashObj* pHashObj = calloc(1, sizeof(SLHashObj));
if (pHashObj == NULL) {
terrno = TSDB_CODE_OUT_OF_MEMORY;
return NULL;
}
int32_t code = createDiskbasedBuf(&pHashObj->pBuf, pageSize, inMemPages * pageSize, 0, "/tmp");
if (code != 0) {
terrno = code;
return NULL;
}
setBufPageCompressOnDisk(pHashObj->pBuf, false);
/**
* The number of bits in the hash value, which is used to decide the exact bucket where the object should be located in.
* The initial value is 0.
*/
pHashObj->bits = 0;
pHashObj->hashFn = fn;
pHashObj->tuplesPerPage = numOfTuplePerPage;
pHashObj->numOfAlloc = 4; // initial allocated array list
pHashObj->pBucket = calloc(pHashObj->numOfAlloc, POINTER_BYTES);
code = doAddNewBucket(pHashObj);
if (code != TSDB_CODE_SUCCESS) {
destroyDiskbasedBuf(pHashObj->pBuf);
tfree(pHashObj);
terrno = code;
return NULL;
}
return pHashObj;
}
void* tHashCleanup(SLHashObj* pHashObj) {
destroyDiskbasedBuf(pHashObj->pBuf);
for(int32_t i = 0; i < pHashObj->numOfBuckets; ++i) {
taosArrayDestroy(pHashObj->pBucket[i]->pPageIdList);
tfree(pHashObj->pBucket[i]);
}
tfree(pHashObj->pBucket);
tfree(pHashObj);
return NULL;
}
int32_t tHashPut(SLHashObj* pHashObj, const void *key, size_t keyLen, void *data, size_t size) {
ASSERT(pHashObj != NULL && key != NULL);
if (pHashObj->bits == 0) {
SLHashBucket* pBucket = pHashObj->pBucket[0];
doAddToBucket(pHashObj, pBucket, 0, key, keyLen, data, size);
} else {
int32_t hashVal = pHashObj->hashFn(key, keyLen);
int32_t v = doGetBucketIdFromHashVal(hashVal, pHashObj->bits);
if (v >= pHashObj->numOfBuckets) {
int32_t newBucketId = doGetAlternativeBucketId(v, pHashObj->bits, pHashObj->numOfBuckets);
// printf("bucketId: 0x%x not exists, put it into 0x%x instead\n", v, newBucketId);
v = newBucketId;
}
SLHashBucket* pBucket = pHashObj->pBucket[v];
int32_t code = doAddToBucket(pHashObj, pBucket, v, key, keyLen, data, size);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
}
pHashObj->size += 1;
// Too many records, needs to bucket split
if ((pHashObj->numOfBuckets * LHASH_CAP_RATIO * pHashObj->tuplesPerPage) < pHashObj->size) {
int32_t newBucketId = pHashObj->numOfBuckets;
int32_t code = doAddNewBucket(pHashObj);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
int32_t numOfBits = ceil(log(pHashObj->numOfBuckets) / log(2));
if (numOfBits > pHashObj->bits) {
// printf("extend the bits from %d to %d, new bucket:%d\n", pHashObj->bits, numOfBits, newBucketId);
ASSERT(numOfBits == pHashObj->bits + 1);
pHashObj->bits = numOfBits;
}
int32_t splitBucketId = doGetRelatedSplitBucketId(newBucketId, pHashObj->bits);
// load all data in this bucket and check if the data needs to relocated into the new bucket
SLHashBucket* pBucket = pHashObj->pBucket[splitBucketId];
// printf("split %d items' bucket:0x%x to new bucket:0x%x\n", pBucket->size, splitBucketId, newBucketId);
for (int32_t i = 0; i < taosArrayGetSize(pBucket->pPageIdList); ++i) {
int32_t pageId = *(int32_t*)taosArrayGet(pBucket->pPageIdList, i);
SFilePage* p = getBufPage(pHashObj->pBuf, pageId);
char* pStart = p->data;
while (pStart - ((char*) p) < p->num) {
SLHashNode* pNode = (SLHashNode*)pStart;
ASSERT(pNode->keyLen > 0 && pNode->dataLen >= 0);
char* k = GET_LHASH_NODE_KEY(pNode);
int32_t hashv = pHashObj->hashFn(k, pNode->keyLen);
int32_t v1 = doGetBucketIdFromHashVal(hashv, pHashObj->bits);
if (v1 != splitBucketId) { // place it into the new bucket
ASSERT(v1 == newBucketId);
// printf("move key:%d to 0x%x bucket, remain items:%d\n", *(int32_t*)k, v1, pBucket->size - 1);
SLHashBucket* pNewBucket = pHashObj->pBucket[newBucketId];
doAddToBucket(pHashObj, pNewBucket, newBucketId, (void*)GET_LHASH_NODE_KEY(pNode), pNode->keyLen,
GET_LHASH_NODE_KEY(pNode), pNode->dataLen);
doRemoveFromBucket(p, pNode, pBucket);
} else {
// printf("check key:%d, located into: %d, skip it\n", *(int*) k, v1);
int32_t nodeSize = GET_LHASH_NODE_LEN(pStart);
pStart += nodeSize;
}
}
releaseBufPage(pHashObj->pBuf, p);
}
doCompressBucketPages(pHashObj, pBucket);
}
return TSDB_CODE_SUCCESS;
}
char* tHashGet(SLHashObj* pHashObj, const void *key, size_t keyLen) {
ASSERT(pHashObj != NULL && key != NULL && keyLen > 0);
int32_t hashv = pHashObj->hashFn(key, keyLen);
int32_t bucketId = doGetBucketIdFromHashVal(hashv, pHashObj->bits);
if (bucketId >= pHashObj->numOfBuckets) {
bucketId = doGetAlternativeBucketId(bucketId, pHashObj->bits, pHashObj->numOfBuckets);
}
SLHashBucket* pBucket = pHashObj->pBucket[bucketId];
for (int32_t i = 0; i < taosArrayGetSize(pBucket->pPageIdList); ++i) {
int32_t pageId = *(int32_t*)taosArrayGet(pBucket->pPageIdList, i);
SFilePage* p = getBufPage(pHashObj->pBuf, pageId);
char* pStart = p->data;
while (pStart - p->data < p->num) {
SLHashNode* pNode = (SLHashNode*)pStart;
char* k = GET_LHASH_NODE_KEY(pNode);
if (pNode->keyLen == keyLen && (memcmp(key, k, keyLen) == 0)) {
releaseBufPage(pHashObj->pBuf, p);
return GET_LHASH_NODE_DATA(pNode);
} else {
pStart += GET_LHASH_NODE_LEN(pStart);
}
}
releaseBufPage(pHashObj->pBuf, p);
}
return NULL;
}
int32_t tHashRemove(SLHashObj* pHashObj, const void *key, size_t keyLen) {
// todo
}
void tHashPrint(const SLHashObj* pHashObj, int32_t type) {
printf("==================== linear hash ====================\n");
printf("total bucket:%d, size:%ld, ratio:%.2f\n", pHashObj->numOfBuckets, pHashObj->size, LHASH_CAP_RATIO);
dBufSetPrintInfo(pHashObj->pBuf);
if (type == LINEAR_HASH_DATA) {
for (int32_t i = 0; i < pHashObj->numOfBuckets; ++i) {
// printf("bucket: 0x%x, obj:%d, page:%d\n", i, pHashObj->pBucket[i]->size,
// (int)taosArrayGetSize(pHashObj->pBucket[i]->pPageIdList));
}
} else {
dBufPrintStatis(pHashObj->pBuf);
}
}

View File

@ -0,0 +1,679 @@
/*
* Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
*
* This program is free software: you can use, redistribute, and/or modify
* it under the terms of the GNU Affero General Public License, version 3
* or later ("AGPL"), as published by the Free Software Foundation.
*
* This program is distributed in the hope that it will be useful, but WITHOUT
* ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
* FITNESS FOR A PARTICULAR PURPOSE.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
#include "common.h"
#include "query.h"
#include "tsort.h"
#include "tep.h"
#include "tcfg.h"
#include "tlosertree.h"
#include "tpagedbuf.h"
#include "tutil.h"
typedef struct STupleHandle {
SSDataBlock* pBlock;
int32_t rowIndex;
} STupleHandle;
typedef struct SSortHandle {
int32_t type;
int32_t pageSize;
int32_t numOfPages;
SDiskbasedBuf *pBuf;
SArray *pOrderInfo;
bool nullFirst;
bool hasVarCol;
SArray *pOrderedSource;
_sort_fetch_block_fn_t fetchfp;
_sort_merge_compar_fn_t comparFn;
void *pParam;
SMultiwayMergeTreeInfo *pMergeTree;
int32_t numOfCols;
int64_t startTs;
uint64_t sortElapsed;
uint64_t totalElapsed;
int32_t sourceId;
SSDataBlock *pDataBlock;
SMsortComparParam cmpParam;
int32_t numOfCompletedSources;
bool opened;
const char *idStr;
bool inMemSort;
bool needAdjust;
STupleHandle tupleHandle;
} SSortHandle;
static int32_t msortComparFn(const void *pLeft, const void *pRight, void *param);
static SSDataBlock* createDataBlock_rv(SSchema* pSchema, int32_t numOfCols) {
SSDataBlock* pBlock = calloc(1, sizeof(SSDataBlock));
pBlock->pDataBlock = taosArrayInit(numOfCols, sizeof(SColumnInfoData));
pBlock->info.numOfCols = numOfCols;
for(int32_t i = 0; i < numOfCols; ++i) {
SColumnInfoData colInfo = {0};
colInfo.info.type = pSchema[i].type;
colInfo.info.bytes = pSchema[i].bytes;
colInfo.info.colId = pSchema[i].colId;
taosArrayPush(pBlock->pDataBlock, &colInfo);
}
return pBlock;
}
/**
*
* @param type
* @return
*/
SSortHandle* tsortCreateSortHandle(SArray* pOrderInfo, bool nullFirst, int32_t type, int32_t pageSize, int32_t numOfPages, SSchema* pSchema, int32_t numOfCols, const char* idstr) {
SSortHandle* pSortHandle = calloc(1, sizeof(SSortHandle));
pSortHandle->type = type;
pSortHandle->pageSize = pageSize;
pSortHandle->numOfPages = numOfPages;
pSortHandle->pOrderedSource = taosArrayInit(4, POINTER_BYTES);
pSortHandle->pOrderInfo = pOrderInfo;
pSortHandle->nullFirst = nullFirst;
pSortHandle->cmpParam.orderInfo = pOrderInfo;
pSortHandle->pDataBlock = createDataBlock_rv(pSchema, numOfCols);
tsortSetComparFp(pSortHandle, msortComparFn);
if (idstr != NULL) {
pSortHandle->idStr = strdup(idstr);
}
return pSortHandle;
}
void tsortDestroySortHandle(SSortHandle* pSortHandle) {
tsortClose(pSortHandle);
if (pSortHandle->pMergeTree != NULL) {
tMergeTreeDestroy(pSortHandle->pMergeTree);
}
destroyDiskbasedBuf(pSortHandle->pBuf);
tfree(pSortHandle->idStr);
tfree(pSortHandle);
}
int32_t tsortAddSource(SSortHandle* pSortHandle, void* pSource) {
taosArrayPush(pSortHandle->pOrderedSource, &pSource);
}
static int32_t doAddNewExternalMemSource(SDiskbasedBuf *pBuf, SArray* pAllSources, SSDataBlock* pBlock, int32_t* sourceId) {
SExternalMemSource* pSource = calloc(1, sizeof(SExternalMemSource));
if (pSource == NULL) {
return TSDB_CODE_QRY_OUT_OF_MEMORY;
}
pSource->pageIdList = getDataBufPagesIdList(pBuf, (*sourceId));
pSource->src.pBlock = pBlock;
taosArrayPush(pAllSources, &pSource);
(*sourceId) += 1;
int32_t rowSize = blockDataGetSerialRowSize(pSource->src.pBlock);
int32_t numOfRows = (getBufPageSize(pBuf) - blockDataGetSerialMetaSize(pBlock))/rowSize;
return blockDataEnsureCapacity(pSource->src.pBlock, numOfRows);
}
static int32_t doAddToBuf(SSDataBlock* pDataBlock, SSortHandle* pHandle) {
int32_t start = 0;
if (pHandle->pBuf == NULL) {
int32_t code = createDiskbasedBuf(&pHandle->pBuf, pHandle->pageSize, pHandle->numOfPages * pHandle->pageSize, 0, "/tmp");
dBufSetPrintInfo(pHandle->pBuf);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
}
while(start < pDataBlock->info.rows) {
int32_t stop = 0;
blockDataSplitRows(pDataBlock, pHandle->hasVarCol, start, &stop, pHandle->pageSize);
SSDataBlock* p = blockDataExtractBlock(pDataBlock, start, stop - start + 1);
if (p == NULL) {
return terrno;
}
int32_t pageId = -1;
SFilePage* pPage = getNewBufPage(pHandle->pBuf, pHandle->sourceId, &pageId);
if (pPage == NULL) {
return terrno;
}
int32_t size = blockDataGetSize(p) + sizeof(int32_t) + p->info.numOfCols * sizeof(int32_t);
assert(size <= getBufPageSize(pHandle->pBuf));
blockDataToBuf(pPage->data, p);
setBufPageDirty(pPage, true);
releaseBufPage(pHandle->pBuf, pPage);
blockDataDestroy(p);
start = stop + 1;
}
blockDataClearup(pDataBlock, pHandle->hasVarCol);
SSDataBlock* pBlock = createOneDataBlock(pDataBlock);
int32_t code = doAddNewExternalMemSource(pHandle->pBuf, pHandle->pOrderedSource, pBlock, &pHandle->sourceId);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
}
static int32_t sortComparInit(SMsortComparParam* cmpParam, SArray* pSources, int32_t startIndex, int32_t endIndex, SSortHandle* pHandle) {
cmpParam->pSources = taosArrayGet(pSources, startIndex);
cmpParam->numOfSources = (endIndex - startIndex + 1);
int32_t code = 0;
if (pHandle->type == SORT_SINGLESOURCE_SORT) {
for (int32_t i = 0; i < cmpParam->numOfSources; ++i) {
SExternalMemSource* pSource = cmpParam->pSources[i];
SPageInfo* pPgInfo = *(SPageInfo**)taosArrayGet(pSource->pageIdList, pSource->pageIndex);
SFilePage* pPage = getBufPage(pHandle->pBuf, getPageId(pPgInfo));
code = blockDataFromBuf(pSource->src.pBlock, pPage->data);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
releaseBufPage(pHandle->pBuf, pPage);
}
} else {
// multi-pass internal merge sort is required
if (pHandle->pBuf == NULL) {
code = createDiskbasedBuf(&pHandle->pBuf, pHandle->pageSize, pHandle->numOfPages * pHandle->pageSize, 0, "/tmp");
dBufSetPrintInfo(pHandle->pBuf);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
}
for (int32_t i = 0; i < cmpParam->numOfSources; ++i) {
SGenericSource* pSource = cmpParam->pSources[i];
pSource->src.pBlock = pHandle->fetchfp(pSource->param);
}
}
return code;
}
static int32_t sortComparClearup(SMsortComparParam* cmpParam) {
for(int32_t i = 0; i < cmpParam->numOfSources; ++i) {
SExternalMemSource* pSource = cmpParam->pSources[i];
blockDataDestroy(pSource->src.pBlock);
tfree(pSource);
}
cmpParam->numOfSources = 0;
}
static void appendOneRowToDataBlock(SSDataBlock *pBlock, const SSDataBlock* pSource, int32_t* rowIndex) {
for (int32_t i = 0; i < pBlock->info.numOfCols; ++i) {
SColumnInfoData* pColInfo = taosArrayGet(pBlock->pDataBlock, i);
SColumnInfoData* pSrcColInfo = taosArrayGet(pSource->pDataBlock, i);
bool isNull = colDataIsNull(pSrcColInfo, pSource->info.rows, *rowIndex, NULL);
if (isNull) {
colDataAppend(pColInfo, pBlock->info.rows, NULL, true);
} else {
char* pData = colDataGetData(pSrcColInfo, *rowIndex);
colDataAppend(pColInfo, pBlock->info.rows, pData, false);
}
}
pBlock->info.rows += 1;
*rowIndex += 1;
}
static int32_t adjustMergeTreeForNextTuple(SExternalMemSource *pSource, SMultiwayMergeTreeInfo *pTree, SSortHandle *pHandle, int32_t* numOfCompleted) {
/*
* load a new SDataBlock into memory of a given intermediate data-set source,
* since it's last record in buffer has been chosen to be processed, as the winner of loser-tree
*/
if (pSource->src.rowIndex >= pSource->src.pBlock->info.rows) {
pSource->src.rowIndex = 0;
if (pHandle->type == SORT_SINGLESOURCE_SORT) {
if (pSource->pageIndex >= taosArrayGetSize(pSource->pageIdList)) {
(*numOfCompleted) += 1;
pSource->src.rowIndex = -1;
pSource->pageIndex = -1;
pSource->src.pBlock = blockDataDestroy(pSource->src.pBlock);
} else {
SPageInfo* pPgInfo = *(SPageInfo**)taosArrayGet(pSource->pageIdList, pSource->pageIndex);
SFilePage* pPage = getBufPage(pHandle->pBuf, getPageId(pPgInfo));
int32_t code = blockDataFromBuf(pSource->src.pBlock, pPage->data);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
releaseBufPage(pHandle->pBuf, pPage);
}
} else {
pSource->src.pBlock = pHandle->fetchfp(((SGenericSource*)pSource)->param);
if (pSource->src.pBlock == NULL) {
(*numOfCompleted) += 1;
pSource->src.rowIndex = -1;
}
}
}
/*
* Adjust loser tree otherwise, according to new candidate data
* if the loser tree is rebuild completed, we do not need to adjust
*/
int32_t leafNodeIndex = tMergeTreeGetAdjustIndex(pTree);
#ifdef _DEBUG_VIEW
printf("before adjust:\t");
tMergeTreePrint(pTree);
#endif
tMergeTreeAdjust(pTree, leafNodeIndex);
#ifdef _DEBUG_VIEW
printf("\nafter adjust:\t");
tMergeTreePrint(pTree);
#endif
}
static SSDataBlock* getSortedBlockData(SSortHandle* pHandle, SMsortComparParam* cmpParam, int32_t capacity) {
blockDataClearup(pHandle->pDataBlock, pHandle->hasVarCol);
while(1) {
if (cmpParam->numOfSources == pHandle->numOfCompletedSources) {
break;
}
int32_t index = tMergeTreeGetChosenIndex(pHandle->pMergeTree);
SExternalMemSource *pSource = (*cmpParam).pSources[index];
appendOneRowToDataBlock(pHandle->pDataBlock, pSource->src.pBlock, &pSource->src.rowIndex);
int32_t code = adjustMergeTreeForNextTuple(pSource, pHandle->pMergeTree, pHandle, &pHandle->numOfCompletedSources);
if (code != TSDB_CODE_SUCCESS) {
terrno = code;
return NULL;
}
if (pHandle->pDataBlock->info.rows >= capacity) {
return pHandle->pDataBlock;
}
}
return (pHandle->pDataBlock->info.rows > 0)? pHandle->pDataBlock:NULL;
}
int32_t msortComparFn(const void *pLeft, const void *pRight, void *param) {
int32_t pLeftIdx = *(int32_t *)pLeft;
int32_t pRightIdx = *(int32_t *)pRight;
SMsortComparParam *pParam = (SMsortComparParam *)param;
SArray *pInfo = pParam->orderInfo;
SExternalMemSource* pLeftSource = pParam->pSources[pLeftIdx];
SExternalMemSource* pRightSource = pParam->pSources[pRightIdx];
// this input is exhausted, set the special value to denote this
if (pLeftSource->src.rowIndex == -1) {
return 1;
}
if (pRightSource->src.rowIndex == -1) {
return -1;
}
SSDataBlock* pLeftBlock = pLeftSource->src.pBlock;
SSDataBlock* pRightBlock = pRightSource->src.pBlock;
for(int32_t i = 0; i < pInfo->size; ++i) {
SBlockOrderInfo* pOrder = TARRAY_GET_ELEM(pInfo, i);
SColumnInfoData* pLeftColInfoData = TARRAY_GET_ELEM(pLeftBlock->pDataBlock, pOrder->colIndex);
bool leftNull = false;
if (pLeftColInfoData->hasNull) {
leftNull = colDataIsNull(pLeftColInfoData, pLeftBlock->info.rows, pLeftSource->src.rowIndex, pLeftBlock->pBlockAgg);
}
SColumnInfoData* pRightColInfoData = TARRAY_GET_ELEM(pRightBlock->pDataBlock, pOrder->colIndex);
bool rightNull = false;
if (pRightColInfoData->hasNull) {
rightNull = colDataIsNull(pRightColInfoData, pRightBlock->info.rows, pRightSource->src.rowIndex, pRightBlock->pBlockAgg);
}
if (leftNull && rightNull) {
continue; // continue to next slot
}
if (rightNull) {
return pParam->nullFirst? 1:-1;
}
if (leftNull) {
return pParam->nullFirst? -1:1;
}
void* left1 = colDataGetData(pLeftColInfoData, pLeftSource->src.rowIndex);
void* right1 = colDataGetData(pRightColInfoData, pRightSource->src.rowIndex);
switch(pLeftColInfoData->info.type) {
case TSDB_DATA_TYPE_INT: {
int32_t leftv = *(int32_t*)left1;
int32_t rightv = *(int32_t*)right1;
if (leftv == rightv) {
break;
} else {
if (pOrder->order == TSDB_ORDER_ASC) {
return leftv < rightv? -1 : 1;
} else {
return leftv < rightv? 1 : -1;
}
}
}
default:
assert(0);
}
}
}
static int32_t doInternalMergeSort(SSortHandle* pHandle) {
size_t numOfSources = taosArrayGetSize(pHandle->pOrderedSource);
// Calculate the I/O counts to complete the data sort.
double sortPass = floorl(log2(numOfSources) / log2(pHandle->numOfPages));
pHandle->totalElapsed = taosGetTimestampUs() - pHandle->startTs;
qDebug("%s %d rounds mergesort required to complete the sort, first-round sorted data size:%"PRIzu", sort:%"PRId64", total elapsed:%"PRId64,
pHandle->idStr, (int32_t) (sortPass + 1), getTotalBufSize(pHandle->pBuf), pHandle->sortElapsed, pHandle->totalElapsed);
size_t pgSize = pHandle->pageSize;
int32_t numOfRows = (pgSize - blockDataGetSerialMetaSize(pHandle->pDataBlock))/ blockDataGetSerialRowSize(pHandle->pDataBlock);
blockDataEnsureCapacity(pHandle->pDataBlock, numOfRows);
size_t numOfSorted = taosArrayGetSize(pHandle->pOrderedSource);
for(int32_t t = 0; t < sortPass; ++t) {
int64_t st = taosGetTimestampUs();
SArray* pResList = taosArrayInit(4, POINTER_BYTES);
int32_t numOfInputSources = pHandle->numOfPages;
int32_t sortGroup = (numOfSorted + numOfInputSources - 1) / numOfInputSources;
// Only *numOfInputSources* can be loaded into buffer to perform the external sort.
for(int32_t i = 0; i < sortGroup; ++i) {
pHandle->sourceId += 1;
int32_t end = (i + 1) * numOfInputSources - 1;
if (end > numOfSorted - 1) {
end = numOfSorted - 1;
}
pHandle->cmpParam.numOfSources = end - i * numOfInputSources + 1;
int32_t code = sortComparInit(&pHandle->cmpParam, pHandle->pOrderedSource, i * numOfInputSources, end, pHandle);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
code = tMergeTreeCreate(&pHandle->pMergeTree, pHandle->cmpParam.numOfSources, &pHandle->cmpParam, pHandle->comparFn);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
while (1) {
SSDataBlock* pDataBlock = getSortedBlockData(pHandle, &pHandle->cmpParam, numOfRows);
if (pDataBlock == NULL) {
break;
}
int32_t pageId = -1;
SFilePage* pPage = getNewBufPage(pHandle->pBuf, pHandle->sourceId, &pageId);
if (pPage == NULL) {
return terrno;
}
int32_t size = blockDataGetSize(pDataBlock) + sizeof(int32_t) + pDataBlock->info.numOfCols * sizeof(int32_t);
assert(size <= getBufPageSize(pHandle->pBuf));
blockDataToBuf(pPage->data, pDataBlock);
setBufPageDirty(pPage, true);
releaseBufPage(pHandle->pBuf, pPage);
blockDataClearup(pDataBlock, pHandle->hasVarCol);
}
tMergeTreeDestroy(pHandle->pMergeTree);
pHandle->numOfCompletedSources = 0;
SSDataBlock* pBlock = createOneDataBlock(pHandle->pDataBlock);
code = doAddNewExternalMemSource(pHandle->pBuf, pResList, pBlock, &pHandle->sourceId);
if (code != 0) {
return code;
}
}
sortComparClearup(&pHandle->cmpParam);
taosArrayClear(pHandle->pOrderedSource);
taosArrayAddAll(pHandle->pOrderedSource, pResList);
taosArrayDestroy(pResList);
numOfSorted = taosArrayGetSize(pHandle->pOrderedSource);
int64_t el = taosGetTimestampUs() - st;
pHandle->totalElapsed += el;
SDiskbasedBufStatis statis = getDBufStatis(pHandle->pBuf);
qDebug("%s %d round mergesort, elapsed:%"PRId64" readDisk:%.2f Kb, flushDisk:%.2f Kb", pHandle->idStr, t + 1, el, statis.loadBytes/1024.0,
statis.flushBytes/1024.0);
if (pHandle->type == SORT_MULTISOURCE_MERGE) {
pHandle->type = SORT_SINGLESOURCE_SORT;
pHandle->comparFn = msortComparFn;
}
}
pHandle->cmpParam.numOfSources = taosArrayGetSize(pHandle->pOrderedSource);
return 0;
}
static int32_t createInitialSortedMultiSources(SSortHandle* pHandle) {
size_t sortBufSize = pHandle->numOfPages * pHandle->pageSize;
if (pHandle->type == SORT_SINGLESOURCE_SORT) {
SGenericSource* source = taosArrayGetP(pHandle->pOrderedSource, 0);
taosArrayClear(pHandle->pOrderedSource);
while (1) {
SSDataBlock* pBlock = pHandle->fetchfp(source->param);
if (pBlock == NULL) {
break;
}
if (pHandle->pDataBlock == NULL) {
pHandle->pDataBlock = createOneDataBlock(pBlock);
}
int32_t code = blockDataMerge(pHandle->pDataBlock, pBlock);
if (code != 0) {
return code;
}
size_t size = blockDataGetSize(pHandle->pDataBlock);
if (size > sortBufSize) {
// Perform the in-memory sort and then flush data in the buffer into disk.
int64_t p = taosGetTimestampUs();
blockDataSort(pHandle->pDataBlock, pHandle->pOrderInfo, pHandle->nullFirst);
int64_t el = taosGetTimestampUs() - p;
pHandle->sortElapsed += el;
doAddToBuf(pHandle->pDataBlock, pHandle);
}
}
if (pHandle->pDataBlock->info.rows > 0) {
size_t size = blockDataGetSize(pHandle->pDataBlock);
// Perform the in-memory sort and then flush data in the buffer into disk.
blockDataSort(pHandle->pDataBlock, pHandle->pOrderInfo, pHandle->nullFirst);
// All sorted data can fit in memory, external memory sort is not needed. Return to directly
if (size <= sortBufSize) {
pHandle->cmpParam.numOfSources = 1;
pHandle->inMemSort = true;
pHandle->tupleHandle.rowIndex = -1;
pHandle->tupleHandle.pBlock = pHandle->pDataBlock;
return 0;
} else {
doAddToBuf(pHandle->pDataBlock, pHandle);
}
}
tfree(source);
}
return TSDB_CODE_SUCCESS;
}
int32_t tsortOpen(SSortHandle* pHandle) {
if (pHandle->opened) {
return 0;
}
if (pHandle->fetchfp == NULL || pHandle->comparFn == NULL) {
return -1;
}
pHandle->opened = true;
int32_t code = createInitialSortedMultiSources(pHandle);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
// do internal sort
code = doInternalMergeSort(pHandle);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
int32_t numOfSources = taosArrayGetSize(pHandle->pOrderedSource);
if (pHandle->pBuf != NULL) {
ASSERT(numOfSources <= getNumOfInMemBufPages(pHandle->pBuf));
}
code = sortComparInit(&pHandle->cmpParam, pHandle->pOrderedSource, 0, numOfSources - 1, pHandle);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
code = tMergeTreeCreate(&pHandle->pMergeTree, pHandle->cmpParam.numOfSources, &pHandle->cmpParam, pHandle->comparFn);
if (code != TSDB_CODE_SUCCESS) {
return code;
}
}
int32_t tsortClose(SSortHandle* pHandle) {
// do nothing
}
int32_t tsortSetFetchRawDataFp(SSortHandle* pHandle, _sort_fetch_block_fn_t fp) {
pHandle->fetchfp = fp;
}
int32_t tsortSetComparFp(SSortHandle* pHandle, _sort_merge_compar_fn_t fp) {
pHandle->comparFn = fp;
}
STupleHandle* tsortNextTuple(SSortHandle* pHandle) {
if (pHandle->cmpParam.numOfSources == pHandle->numOfCompletedSources) {
return NULL;
}
// All the data are hold in the buffer, no external sort is invoked.
if (pHandle->inMemSort) {
pHandle->tupleHandle.rowIndex += 1;
if (pHandle->tupleHandle.rowIndex == pHandle->pDataBlock->info.rows) {
pHandle->numOfCompletedSources = 1;
return NULL;
}
return &pHandle->tupleHandle;
}
int32_t index = tMergeTreeGetChosenIndex(pHandle->pMergeTree);
SExternalMemSource *pSource = pHandle->cmpParam.pSources[index];
if (pHandle->needAdjust) {
int32_t code = adjustMergeTreeForNextTuple(pSource, pHandle->pMergeTree, pHandle, &pHandle->numOfCompletedSources);
if (code != TSDB_CODE_SUCCESS) {
terrno = code;
return NULL;
}
}
if (pHandle->cmpParam.numOfSources == pHandle->numOfCompletedSources) {
return NULL;
}
// Get the adjusted value after the loser tree is updated.
index = tMergeTreeGetChosenIndex(pHandle->pMergeTree);
pSource = pHandle->cmpParam.pSources[index];
assert(pSource->src.pBlock != NULL);
pHandle->tupleHandle.rowIndex = pSource->src.rowIndex;
pHandle->tupleHandle.pBlock = pSource->src.pBlock;
pHandle->needAdjust = true;
pSource->src.rowIndex += 1;
return &pHandle->tupleHandle;
}
bool tsortIsNullVal(STupleHandle* pVHandle, int32_t colIndex) {
return false;
}
void* tsortGetValue(STupleHandle* pVHandle, int32_t colIndex) {
SColumnInfoData* pColInfo = TARRAY_GET_ELEM(pVHandle->pBlock->pDataBlock, colIndex);
return colDataGetData(pColInfo, pVHandle->rowIndex);
}

View File

@ -14,6 +14,7 @@
*/ */
#include <executorimpl.h> #include <executorimpl.h>
#include <function.h>
#include <gtest/gtest.h> #include <gtest/gtest.h>
#include <tglobal.h> #include <tglobal.h>
#include <iostream> #include <iostream>
@ -37,22 +38,30 @@
namespace { namespace {
enum {
data_rand = 0x1,
data_asc = 0x2,
data_desc = 0x3,
};
typedef struct SDummyInputInfo { typedef struct SDummyInputInfo {
int32_t max; int32_t totalPages; // numOfPages
int32_t current; int32_t current;
int32_t startVal; int32_t startVal;
int32_t type;
int32_t numOfRowsPerPage;
int32_t numOfCols; // number of columns
int64_t tsStart;
SSDataBlock* pBlock; SSDataBlock* pBlock;
} SDummyInputInfo; } SDummyInputInfo;
SSDataBlock* getDummyBlock(void* param, bool* newgroup) { SSDataBlock* getDummyBlock(void* param, bool* newgroup) {
SOperatorInfo* pOperator = static_cast<SOperatorInfo*>(param); SOperatorInfo* pOperator = static_cast<SOperatorInfo*>(param);
SDummyInputInfo* pInfo = static_cast<SDummyInputInfo*>(pOperator->info); SDummyInputInfo* pInfo = static_cast<SDummyInputInfo*>(pOperator->info);
if (pInfo->current >= pInfo->max) { if (pInfo->current >= pInfo->totalPages) {
return NULL; return NULL;
} }
int32_t numOfRows = 1000;
if (pInfo->pBlock == NULL) { if (pInfo->pBlock == NULL) {
pInfo->pBlock = static_cast<SSDataBlock*>(calloc(1, sizeof(SSDataBlock))); pInfo->pBlock = static_cast<SSDataBlock*>(calloc(1, sizeof(SSDataBlock)));
@ -62,8 +71,8 @@ SSDataBlock* getDummyBlock(void* param, bool* newgroup) {
colInfo.info.type = TSDB_DATA_TYPE_INT; colInfo.info.type = TSDB_DATA_TYPE_INT;
colInfo.info.bytes = sizeof(int32_t); colInfo.info.bytes = sizeof(int32_t);
colInfo.info.colId = 1; colInfo.info.colId = 1;
colInfo.pData = static_cast<char*>(calloc(numOfRows, sizeof(int32_t))); colInfo.pData = static_cast<char*>(calloc(pInfo->numOfRowsPerPage, sizeof(int32_t)));
colInfo.nullbitmap = static_cast<char*>(calloc(1, (numOfRows + 7) / 8)); colInfo.nullbitmap = static_cast<char*>(calloc(1, (pInfo->numOfRowsPerPage + 7) / 8));
taosArrayPush(pInfo->pBlock->pDataBlock, &colInfo); taosArrayPush(pInfo->pBlock->pDataBlock, &colInfo);
@ -85,10 +94,18 @@ SSDataBlock* getDummyBlock(void* param, bool* newgroup) {
char buf[128] = {0}; char buf[128] = {0};
char b1[128] = {0}; char b1[128] = {0};
for(int32_t i = 0; i < numOfRows; ++i) { int32_t v = 0;
for(int32_t i = 0; i < pInfo->numOfRowsPerPage; ++i) {
SColumnInfoData* pColInfo = static_cast<SColumnInfoData*>(TARRAY_GET_ELEM(pBlock->pDataBlock, 0)); SColumnInfoData* pColInfo = static_cast<SColumnInfoData*>(TARRAY_GET_ELEM(pBlock->pDataBlock, 0));
int32_t v = (--pInfo->startVal); if (pInfo->type == data_desc) {
v = (--pInfo->startVal);
} else if (pInfo->type == data_asc) {
v = ++pInfo->startVal;
} else if (pInfo->type == data_rand) {
v = random();
}
colDataAppend(pColInfo, i, reinterpret_cast<const char*>(&v), false); colDataAppend(pColInfo, i, reinterpret_cast<const char*>(&v), false);
// sprintf(buf, "this is %d row", i); // sprintf(buf, "this is %d row", i);
@ -98,21 +115,103 @@ SSDataBlock* getDummyBlock(void* param, bool* newgroup) {
// colDataAppend(pColInfo2, i, b1, false); // colDataAppend(pColInfo2, i, b1, false);
} }
pBlock->info.rows = numOfRows; pBlock->info.rows = pInfo->numOfRowsPerPage;
pBlock->info.numOfCols = 1; pBlock->info.numOfCols = 1;
pInfo->current += 1; pInfo->current += 1;
return pBlock; return pBlock;
} }
SOperatorInfo* createDummyOperator(int32_t numOfBlocks) { SSDataBlock* get2ColsDummyBlock(void* param, bool* newgroup) {
SOperatorInfo* pOperator = static_cast<SOperatorInfo*>(param);
SDummyInputInfo* pInfo = static_cast<SDummyInputInfo*>(pOperator->info);
if (pInfo->current >= pInfo->totalPages) {
return NULL;
}
if (pInfo->pBlock == NULL) {
pInfo->pBlock = static_cast<SSDataBlock*>(calloc(1, sizeof(SSDataBlock)));
pInfo->pBlock->pDataBlock = taosArrayInit(4, sizeof(SColumnInfoData));
SColumnInfoData colInfo = {0};
colInfo.info.type = TSDB_DATA_TYPE_TIMESTAMP;
colInfo.info.bytes = sizeof(int64_t);
colInfo.info.colId = 1;
colInfo.pData = static_cast<char*>(calloc(pInfo->numOfRowsPerPage, sizeof(int64_t)));
// colInfo.nullbitmap = static_cast<char*>(calloc(1, (pInfo->numOfRowsPerPage + 7) / 8));
taosArrayPush(pInfo->pBlock->pDataBlock, &colInfo);
SColumnInfoData colInfo1 = {0};
colInfo1.info.type = TSDB_DATA_TYPE_INT;
colInfo1.info.bytes = 4;
colInfo1.info.colId = 2;
colInfo1.pData = static_cast<char*>(calloc(pInfo->numOfRowsPerPage, sizeof(int32_t)));
colInfo1.nullbitmap = static_cast<char*>(calloc(1, (pInfo->numOfRowsPerPage + 7) / 8));
taosArrayPush(pInfo->pBlock->pDataBlock, &colInfo1);
} else {
blockDataClearup(pInfo->pBlock, false);
}
SSDataBlock* pBlock = pInfo->pBlock;
char buf[128] = {0};
char b1[128] = {0};
int64_t ts = 0;
int32_t v = 0;
for(int32_t i = 0; i < pInfo->numOfRowsPerPage; ++i) {
SColumnInfoData* pColInfo = static_cast<SColumnInfoData*>(TARRAY_GET_ELEM(pBlock->pDataBlock, 0));
ts = (++pInfo->tsStart);
colDataAppend(pColInfo, i, reinterpret_cast<const char*>(&ts), false);
SColumnInfoData* pColInfo1 = static_cast<SColumnInfoData*>(TARRAY_GET_ELEM(pBlock->pDataBlock, 1));
if (pInfo->type == data_desc) {
v = (--pInfo->startVal);
} else if (pInfo->type == data_asc) {
v = ++pInfo->startVal;
} else if (pInfo->type == data_rand) {
v = random();
}
colDataAppend(pColInfo1, i, reinterpret_cast<const char*>(&v), false);
// sprintf(buf, "this is %d row", i);
// STR_TO_VARSTR(b1, buf);
//
// SColumnInfoData* pColInfo2 = static_cast<SColumnInfoData*>(TARRAY_GET_ELEM(pBlock->pDataBlock, 1));
// colDataAppend(pColInfo2, i, b1, false);
}
pBlock->info.rows = pInfo->numOfRowsPerPage;
pBlock->info.numOfCols = 1;
pInfo->current += 1;
blockDataUpdateTsWindow(pBlock);
return pBlock;
}
SOperatorInfo* createDummyOperator(int32_t startVal, int32_t numOfBlocks, int32_t rowsPerPage, int32_t type, int32_t numOfCols) {
SOperatorInfo* pOperator = static_cast<SOperatorInfo*>(calloc(1, sizeof(SOperatorInfo))); SOperatorInfo* pOperator = static_cast<SOperatorInfo*>(calloc(1, sizeof(SOperatorInfo)));
pOperator->name = "dummyInputOpertor4Test"; pOperator->name = "dummyInputOpertor4Test";
pOperator->exec = getDummyBlock;
if (numOfCols == 1) {
pOperator->exec = getDummyBlock;
} else {
pOperator->exec = get2ColsDummyBlock;
}
SDummyInputInfo *pInfo = (SDummyInputInfo*) calloc(1, sizeof(SDummyInputInfo)); SDummyInputInfo *pInfo = (SDummyInputInfo*) calloc(1, sizeof(SDummyInputInfo));
pInfo->max = numOfBlocks; pInfo->totalPages = numOfBlocks;
pInfo->startVal = 1500000; pInfo->startVal = startVal;
pInfo->numOfRowsPerPage = rowsPerPage;
pInfo->type = type;
pInfo->tsStart = 1620000000000;
pOperator->info = pInfo; pOperator->info = pInfo;
return pOperator; return pOperator;
@ -123,6 +222,7 @@ int main(int argc, char** argv) {
return RUN_ALL_TESTS(); return RUN_ALL_TESTS();
} }
#if 0
TEST(testCase, build_executor_tree_Test) { TEST(testCase, build_executor_tree_Test) {
const char* msg = "{\n" const char* msg = "{\n"
"\t\"Id\":\t{\n" "\t\"Id\":\t{\n"
@ -218,34 +318,34 @@ TEST(testCase, build_executor_tree_Test) {
// int32_t code = qCreateExecTask(&handle, 2, 1, NULL, (void**) &pTaskInfo, &sinkHandle); // int32_t code = qCreateExecTask(&handle, 2, 1, NULL, (void**) &pTaskInfo, &sinkHandle);
} }
//TEST(testCase, inMem_sort_Test) { TEST(testCase, inMem_sort_Test) {
// SArray* pOrderVal = taosArrayInit(4, sizeof(SOrder)); SArray* pOrderVal = taosArrayInit(4, sizeof(SOrder));
// SOrder o = {.order = TSDB_ORDER_ASC}; SOrder o = {.order = TSDB_ORDER_ASC};
// o.col.info.colId = 1; o.col.info.colId = 1;
// o.col.info.type = TSDB_DATA_TYPE_INT; o.col.info.type = TSDB_DATA_TYPE_INT;
// taosArrayPush(pOrderVal, &o); taosArrayPush(pOrderVal, &o);
//
// SArray* pExprInfo = taosArrayInit(4, sizeof(SExprInfo)); SArray* pExprInfo = taosArrayInit(4, sizeof(SExprInfo));
// SExprInfo *exp = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo))); SExprInfo *exp = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo)));
// exp->base.resSchema = createSchema(TSDB_DATA_TYPE_INT, sizeof(int32_t), 1, "res"); exp->base.resSchema = createSchema(TSDB_DATA_TYPE_INT, sizeof(int32_t), 1, "res");
// taosArrayPush(pExprInfo, &exp); taosArrayPush(pExprInfo, &exp);
//
// SExprInfo *exp1 = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo))); SExprInfo *exp1 = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo)));
// exp1->base.resSchema = createSchema(TSDB_DATA_TYPE_BINARY, 40, 2, "res1"); exp1->base.resSchema = createSchema(TSDB_DATA_TYPE_BINARY, 40, 2, "res1");
// taosArrayPush(pExprInfo, &exp1); taosArrayPush(pExprInfo, &exp1);
//
// SOperatorInfo* pOperator = createOrderOperatorInfo(createDummyOperator(5), pExprInfo, pOrderVal); SOperatorInfo* pOperator = createOrderOperatorInfo(createDummyOperator(5), pExprInfo, pOrderVal, NULL);
//
// bool newgroup = false; bool newgroup = false;
// SSDataBlock* pRes = pOperator->exec(pOperator, &newgroup); SSDataBlock* pRes = pOperator->exec(pOperator, &newgroup);
//
// SColumnInfoData* pCol1 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 0)); SColumnInfoData* pCol1 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 0));
// SColumnInfoData* pCol2 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 1)); SColumnInfoData* pCol2 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 1));
// for(int32_t i = 0; i < pRes->info.rows; ++i) { for(int32_t i = 0; i < pRes->info.rows; ++i) {
// char* p = colDataGet(pCol2, i); char* p = colDataGetData(pCol2, i);
// printf("%d: %d, %s\n", i, ((int32_t*)pCol1->pData)[i], (char*)varDataVal(p)); printf("%d: %d, %s\n", i, ((int32_t*)pCol1->pData)[i], (char*)varDataVal(p));
// } }
//} }
typedef struct su { typedef struct su {
int32_t v; int32_t v;
@ -300,7 +400,7 @@ TEST(testCase, external_sort_Test) {
exp1->base.resSchema = createSchema(TSDB_DATA_TYPE_BINARY, 40, 2, "res1"); exp1->base.resSchema = createSchema(TSDB_DATA_TYPE_BINARY, 40, 2, "res1");
// taosArrayPush(pExprInfo, &exp1); // taosArrayPush(pExprInfo, &exp1);
SOperatorInfo* pOperator = createOrderOperatorInfo(createDummyOperator(1500), pExprInfo, pOrderVal); SOperatorInfo* pOperator = createOrderOperatorInfo(createDummyOperator(1500), pExprInfo, pOrderVal, NULL);
bool newgroup = false; bool newgroup = false;
SSDataBlock* pRes = NULL; SSDataBlock* pRes = NULL;
@ -326,14 +426,12 @@ TEST(testCase, external_sort_Test) {
SColumnInfoData* pCol1 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 0)); SColumnInfoData* pCol1 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 0));
// SColumnInfoData* pCol2 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 1)); // SColumnInfoData* pCol2 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 1));
for (int32_t i = 0; i < pRes->info.rows; ++i) { for (int32_t i = 0; i < pRes->info.rows; ++i) {
// char* p = colDataGet(pCol2, i); // char* p = colDataGetData(pCol2, i);
printf("%d: %d\n", total++, ((int32_t*)pCol1->pData)[i]); printf("%d: %d\n", total++, ((int32_t*)pCol1->pData)[i]);
// printf("%d: %d, %s\n", total++, ((int32_t*)pCol1->pData)[i], (char*)varDataVal(p)); // printf("%d: %d, %s\n", total++, ((int32_t*)pCol1->pData)[i], (char*)varDataVal(p));
} }
} }
printStatisBeforeClose(((SOrderOperatorInfo*) pOperator->info)->pSortInternalBuf);
int64_t s2 = taosGetTimestampUs(); int64_t s2 = taosGetTimestampUs();
printf("total:%ld\n", s2 - s1); printf("total:%ld\n", s2 - s1);
@ -343,4 +441,153 @@ TEST(testCase, external_sort_Test) {
taosArrayDestroy(pExprInfo); taosArrayDestroy(pExprInfo);
taosArrayDestroy(pOrderVal); taosArrayDestroy(pOrderVal);
} }
TEST(testCase, sorted_merge_Test) {
srand(time(NULL));
SArray* pOrderVal = taosArrayInit(4, sizeof(SOrder));
SOrder o = {0};
o.order = TSDB_ORDER_ASC;
o.col.info.colId = 1;
o.col.info.type = TSDB_DATA_TYPE_INT;
taosArrayPush(pOrderVal, &o);
SArray* pExprInfo = taosArrayInit(4, sizeof(SExprInfo));
SExprInfo *exp = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo)));
exp->base.resSchema = createSchema(TSDB_DATA_TYPE_BIGINT, sizeof(int64_t), 1, "count_result");
exp->base.pColumns = static_cast<SColumn*>(calloc(1, sizeof(SColumn)));
exp->base.pColumns->flag = TSDB_COL_NORMAL;
exp->base.pColumns->info = (SColumnInfo) {.colId = 1, .type = TSDB_DATA_TYPE_INT, .bytes = 4};
exp->base.numOfCols = 1;
taosArrayPush(pExprInfo, &exp);
SExprInfo *exp1 = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo)));
exp1->base.resSchema = createSchema(TSDB_DATA_TYPE_BINARY, 40, 2, "res1");
// taosArrayPush(pExprInfo, &exp1);
int32_t numOfSources = 10;
SOperatorInfo** plist = (SOperatorInfo**) calloc(numOfSources, sizeof(void*));
for(int32_t i = 0; i < numOfSources; ++i) {
plist[i] = createDummyOperator(1, 1, 1, data_asc);
}
SOperatorInfo* pOperator = createSortedMergeOperatorInfo(plist, numOfSources, pExprInfo, pOrderVal, NULL, NULL);
bool newgroup = false;
SSDataBlock* pRes = NULL;
int32_t total = 1;
int64_t s1 = taosGetTimestampUs();
int32_t t = 1;
while(1) {
int64_t s = taosGetTimestampUs();
pRes = pOperator->exec(pOperator, &newgroup);
int64_t e = taosGetTimestampUs();
if (t++ == 1) {
printf("---------------elapsed:%ld\n", e - s);
}
if (pRes == NULL) {
break;
}
SColumnInfoData* pCol1 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 0));
// SColumnInfoData* pCol2 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 1));
for (int32_t i = 0; i < pRes->info.rows; ++i) {
// char* p = colDataGetData(pCol2, i);
printf("%d: %ld\n", total++, ((int64_t*)pCol1->pData)[i]);
// printf("%d: %d, %s\n", total++, ((int32_t*)pCol1->pData)[i], (char*)varDataVal(p));
}
}
int64_t s2 = taosGetTimestampUs();
printf("total:%ld\n", s2 - s1);
pOperator->cleanupFn(pOperator->info, 2);
tfree(exp);
tfree(exp1);
taosArrayDestroy(pExprInfo);
taosArrayDestroy(pOrderVal);
}
#endif
TEST(testCase, time_interval_Operator_Test) {
srand(time(NULL));
SArray* pOrderVal = taosArrayInit(4, sizeof(SOrder));
SOrder o = {0};
o.order = TSDB_ORDER_ASC;
o.col.info.colId = 1;
o.col.info.type = TSDB_DATA_TYPE_INT;
taosArrayPush(pOrderVal, &o);
SArray* pExprInfo = taosArrayInit(4, sizeof(SExprInfo));
SExprInfo *exp = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo)));
exp->base.resSchema = createSchema(TSDB_DATA_TYPE_TIMESTAMP, sizeof(int64_t), 1, "ts");
exp->base.pColumns = static_cast<SColumn*>(calloc(1, sizeof(SColumn)));
exp->base.pColumns->flag = TSDB_COL_NORMAL;
exp->base.pColumns->info = (SColumnInfo) {.colId = 1, .type = TSDB_DATA_TYPE_TIMESTAMP, .bytes = 8};
exp->base.numOfCols = 1;
taosArrayPush(pExprInfo, &exp);
SExprInfo *exp1 = static_cast<SExprInfo*>(calloc(1, sizeof(SExprInfo)));
exp1->base.resSchema = createSchema(TSDB_DATA_TYPE_BIGINT, 8, 2, "res1");
exp1->base.pColumns = static_cast<SColumn*>(calloc(1, sizeof(SColumn)));
exp1->base.pColumns->flag = TSDB_COL_NORMAL;
exp1->base.pColumns->info = (SColumnInfo) {.colId = 1, .type = TSDB_DATA_TYPE_INT, .bytes = 4};
exp1->base.numOfCols = 1;
taosArrayPush(pExprInfo, &exp1);
SOperatorInfo* p = createDummyOperator(1, 1, 2000, data_asc, 2);
SExecTaskInfo ti = {0};
SOperatorInfo* pOperator = createIntervalOperatorInfo(p, pExprInfo, &ti);
bool newgroup = false;
SSDataBlock* pRes = NULL;
int32_t total = 1;
int64_t s1 = taosGetTimestampUs();
int32_t t = 1;
while(1) {
int64_t s = taosGetTimestampUs();
pRes = pOperator->exec(pOperator, &newgroup);
int64_t e = taosGetTimestampUs();
if (t++ == 1) {
printf("---------------elapsed:%ld\n", e - s);
}
if (pRes == NULL) {
break;
}
SColumnInfoData* pCol1 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 0));
// SColumnInfoData* pCol2 = static_cast<SColumnInfoData*>(taosArrayGet(pRes->pDataBlock, 1));
for (int32_t i = 0; i < pRes->info.rows; ++i) {
// char* p = colDataGetData(pCol2, i);
printf("%d: %ld\n", total++, ((int64_t*)pCol1->pData)[i]);
// printf("%d: %d, %s\n", total++, ((int32_t*)pCol1->pData)[i], (char*)varDataVal(p));
}
}
int64_t s2 = taosGetTimestampUs();
printf("total:%ld\n", s2 - s1);
pOperator->cleanupFn(pOperator->info, 2);
tfree(exp);
tfree(exp1);
taosArrayDestroy(pExprInfo);
taosArrayDestroy(pOrderVal);
}
#pragma GCC diagnostic pop #pragma GCC diagnostic pop

View File

@ -0,0 +1,65 @@
/*
* Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
*
* This program is free software: you can use, redistribute, and/or modify
* it under the terms of the GNU Affero General Public License, version 3
* or later ("AGPL"), as published by the Free Software Foundation.
*
* This program is distributed in the hope that it will be useful, but WITHOUT
* ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
* FITNESS FOR A PARTICULAR PURPOSE.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
#include <gtest/gtest.h>
#include <iostream>
#include "executorimpl.h"
#include "tlinearhash.h"
#pragma GCC diagnostic push
#pragma GCC diagnostic ignored "-Wwrite-strings"
#pragma GCC diagnostic ignored "-Wunused-function"
#pragma GCC diagnostic ignored "-Wunused-variable"
#pragma GCC diagnostic ignored "-Wsign-compare"
TEST(testCase, linear_hash_Tests) {
srand(time(NULL));
_hash_fn_t fn = taosGetDefaultHashFunction(TSDB_DATA_TYPE_INT);
#if 0
SLHashObj* pHashObj = tHashInit(256, 4096, fn, 320);
for(int32_t i = 0; i < 5000000; ++i) {
int32_t code = tHashPut(pHashObj, &i, sizeof(i), &i, sizeof(i));
assert(code == 0);
}
// tHashPrint(pHashObj, LINEAR_HASH_STATIS);
// for(int32_t i = 0; i < 10000; ++i) {
// char* v = tHashGet(pHashObj, &i, sizeof(i));
// if (v != NULL) {
//// printf("find value: %d, key:%d\n", *(int32_t*) v, i);
// } else {
// printf("failed to found key:%d in hash\n", i);
// }
// }
tHashPrint(pHashObj, LINEAR_HASH_STATIS);
tHashCleanup(pHashObj);
#endif
#if 0
SHashObj* pHashObj = taosHashInit(1000, fn, false, HASH_NO_LOCK);
for(int32_t i = 0; i < 1000000; ++i) {
taosHashPut(pHashObj, &i, sizeof(i), &i, sizeof(i));
}
for(int32_t i = 0; i < 10000; ++i) {
void* v = taosHashGet(pHashObj, &i, sizeof(i));
}
taosHashCleanup(pHashObj);
#endif
}

View File

@ -0,0 +1,280 @@
/*
* Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
*
* This program is free software: you can use, redistribute, and/or modify
* it under the terms of the GNU Affero General Public License, version 3
* or later ("AGPL"), as published by the Free Software Foundation.
*
* This program is distributed in the hope that it will be useful, but WITHOUT
* ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
* FITNESS FOR A PARTICULAR PURPOSE.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
#include <executorimpl.h>
#include <gtest/gtest.h>
#include <tglobal.h>
#include <tsort.h>
#include <iostream>
#pragma GCC diagnostic push
#pragma GCC diagnostic ignored "-Wwrite-strings"
#pragma GCC diagnostic ignored "-Wunused-function"
#pragma GCC diagnostic ignored "-Wunused-variable"
#pragma GCC diagnostic ignored "-Wsign-compare"
#include "os.h"
#include "executor.h"
#include "stub.h"
#include "taos.h"
#include "tdef.h"
#include "tep.h"
#include "trpc.h"
#include "tvariant.h"
namespace {
typedef struct {
int32_t startVal;
int32_t count;
int32_t pageRows;
} _info;
SSDataBlock* getSingleColDummyBlock(void* param) {
_info* pInfo = (_info*) param;
if (--pInfo->count < 0) {
return NULL;
}
SSDataBlock* pBlock = static_cast<SSDataBlock*>(calloc(1, sizeof(SSDataBlock)));
pBlock->pDataBlock = taosArrayInit(4, sizeof(SColumnInfoData));
SColumnInfoData colInfo = {0};
colInfo.info.type = TSDB_DATA_TYPE_INT;
colInfo.info.bytes = sizeof(int32_t);
colInfo.info.colId = 1;
colInfo.pData = static_cast<char*>(calloc(pInfo->pageRows, sizeof(int32_t)));
colInfo.nullbitmap = static_cast<char*>(calloc(1, (pInfo->pageRows + 7) / 8));
taosArrayPush(pBlock->pDataBlock, &colInfo);
for (int32_t i = 0; i < pInfo->pageRows; ++i) {
SColumnInfoData* pColInfo = static_cast<SColumnInfoData*>(TARRAY_GET_ELEM(pBlock->pDataBlock, 0));
int32_t v = ++pInfo->startVal;
colDataAppend(pColInfo, i, reinterpret_cast<const char*>(&v), false);
}
pBlock->info.rows = pInfo->pageRows;
pBlock->info.numOfCols = 1;
return pBlock;
}
int32_t docomp(const void* p1, const void* p2, void* param) {
int32_t pLeftIdx = *(int32_t *)p1;
int32_t pRightIdx = *(int32_t *)p2;
SMsortComparParam *pParam = (SMsortComparParam *)param;
SGenericSource** px = reinterpret_cast<SGenericSource**>(pParam->pSources);
SArray *pInfo = pParam->orderInfo;
SGenericSource* pLeftSource = px[pLeftIdx];
SGenericSource* pRightSource = px[pRightIdx];
// this input is exhausted, set the special value to denote this
if (pLeftSource->src.rowIndex == -1) {
return 1;
}
if (pRightSource->src.rowIndex == -1) {
return -1;
}
SSDataBlock* pLeftBlock = pLeftSource->src.pBlock;
SSDataBlock* pRightBlock = pRightSource->src.pBlock;
for(int32_t i = 0; i < pInfo->size; ++i) {
SBlockOrderInfo* pOrder = (SBlockOrderInfo*)TARRAY_GET_ELEM(pInfo, i);
SColumnInfoData* pLeftColInfoData = (SColumnInfoData*)TARRAY_GET_ELEM(pLeftBlock->pDataBlock, pOrder->colIndex);
bool leftNull = false;
if (pLeftColInfoData->hasNull) {
leftNull = colDataIsNull(pLeftColInfoData, pLeftBlock->info.rows, pLeftSource->src.rowIndex, pLeftBlock->pBlockAgg);
}
SColumnInfoData* pRightColInfoData = (SColumnInfoData*) TARRAY_GET_ELEM(pRightBlock->pDataBlock, pOrder->colIndex);
bool rightNull = false;
if (pRightColInfoData->hasNull) {
rightNull = colDataIsNull(pRightColInfoData, pRightBlock->info.rows, pRightSource->src.rowIndex, pRightBlock->pBlockAgg);
}
if (leftNull && rightNull) {
continue; // continue to next slot
}
if (rightNull) {
return pParam->nullFirst? 1:-1;
}
if (leftNull) {
return pParam->nullFirst? -1:1;
}
void* left1 = colDataGetData(pLeftColInfoData, pLeftSource->src.rowIndex);
void* right1 = colDataGetData(pRightColInfoData, pRightSource->src.rowIndex);
switch(pLeftColInfoData->info.type) {
case TSDB_DATA_TYPE_INT: {
int32_t leftv = *(int32_t*)left1;
int32_t rightv = *(int32_t*)right1;
if (leftv == rightv) {
break;
} else {
if (pOrder->order == TSDB_ORDER_ASC) {
return leftv < rightv? -1 : 1;
} else {
return leftv < rightv? 1 : -1;
}
}
}
default:
assert(0);
}
}
return 0;
}
} // namespace
#if 0
TEST(testCase, inMem_sort_Test) {
SArray* pOrderVal = taosArrayInit(4, sizeof(SOrder));
SOrder o = {.order = TSDB_ORDER_ASC};
o.col.info.colId = 1;
o.col.info.type = TSDB_DATA_TYPE_INT;
taosArrayPush(pOrderVal, &o);
int32_t numOfRows = 1000;
SBlockOrderInfo oi = {0};
oi.order = TSDB_ORDER_ASC;
oi.colIndex = 0;
SArray* orderInfo = taosArrayInit(1, sizeof(SBlockOrderInfo));
taosArrayPush(orderInfo, &oi);
SSchema s = {.type = TSDB_DATA_TYPE_INT, .colId = 1, .bytes = 4, };
SSortHandle* phandle = tsortCreateSortHandle(orderInfo, false, SORT_SINGLESOURCE_SORT, 1024, 5, &s, 1, "test_abc");
tsortSetFetchRawDataFp(phandle, getSingleColDummyBlock);
tsortAddSource(phandle, &numOfRows);
int32_t code = tsortOpen(phandle);
int32_t row = 1;
while(1) {
STupleHandle* pTupleHandle = tsortNextTuple(phandle);
if (pTupleHandle == NULL) {
break;
}
void* v = tsortGetValue(pTupleHandle, 0);
printf("%d: %d\n", row++, *(int32_t*) v);
}
tsortDestroySortHandle(phandle);
}
TEST(testCase, external_mem_sort_Test) {
SArray* pOrderVal = taosArrayInit(4, sizeof(SOrder));
SOrder o = {.order = TSDB_ORDER_ASC};
o.col.info.colId = 1;
o.col.info.type = TSDB_DATA_TYPE_INT;
taosArrayPush(pOrderVal, &o);
SBlockOrderInfo oi = {0};
oi.order = TSDB_ORDER_ASC;
oi.colIndex = 0;
SArray* orderInfo = taosArrayInit(1, sizeof(SBlockOrderInfo));
taosArrayPush(orderInfo, &oi);
SSchema s = {.type = TSDB_DATA_TYPE_INT, .colId = 1, .bytes = 4, };
SSortHandle* phandle = tsortCreateSortHandle(orderInfo, false, SORT_SINGLESOURCE_SORT, 1024, 5, &s, 1, "test_abc");
tsortSetFetchRawDataFp(phandle, getSingleColDummyBlock);
_info* pInfo = (_info*) calloc(1, sizeof(_info));
pInfo->startVal = 100000;
pInfo->pageRows = 1000;
pInfo->count = 50;
SGenericSource* ps = static_cast<SGenericSource*>(calloc(1, sizeof(SGenericSource)));
ps->param = pInfo;
tsortAddSource(phandle, ps);
int32_t code = tsortOpen(phandle);
int32_t row = 1;
while(1) {
STupleHandle* pTupleHandle = tsortNextTuple(phandle);
if (pTupleHandle == NULL) {
break;
}
void* v = tsortGetValue(pTupleHandle, 0);
printf("%d: %d\n", row++, *(int32_t*) v);
}
tsortDestroySortHandle(phandle);
}
TEST(testCase, ordered_merge_sort_Test) {
SArray* pOrderVal = taosArrayInit(4, sizeof(SOrder));
SOrder o = {.order = TSDB_ORDER_ASC};
o.col.info.colId = 1;
o.col.info.type = TSDB_DATA_TYPE_INT;
taosArrayPush(pOrderVal, &o);
int32_t numOfRows = 1000;
SBlockOrderInfo oi = {0};
oi.order = TSDB_ORDER_ASC;
oi.colIndex = 0;
SArray* orderInfo = taosArrayInit(1, sizeof(SBlockOrderInfo));
taosArrayPush(orderInfo, &oi);
SSchema s = {.type = TSDB_DATA_TYPE_INT, .colId = 1, .bytes = 4};
SSortHandle* phandle = tsortCreateSortHandle(orderInfo, false, SORT_MULTISOURCE_MERGE, 1024, 5, &s, 1,"test_abc");
tsortSetFetchRawDataFp(phandle, getSingleColDummyBlock);
tsortSetComparFp(phandle, docomp);
for(int32_t i = 0; i < 10; ++i) {
SGenericSource* p = static_cast<SGenericSource*>(calloc(1, sizeof(SGenericSource)));
_info* c = static_cast<_info*>(calloc(1, sizeof(_info)));
c->count = 1;
c->pageRows = 1000;
c->startVal = 0;
p->param = c;
tsortAddSource(phandle, p);
}
int32_t code = tsortOpen(phandle);
int32_t row = 1;
while(1) {
STupleHandle* pTupleHandle = tsortNextTuple(phandle);
if (pTupleHandle == NULL) {
break;
}
void* v = tsortGetValue(pTupleHandle, 0);
printf("%d: %d\n", row++, *(int32_t*) v);
}
tsortDestroySortHandle(phandle);
}
#endif
#pragma GCC diagnostic pop

View File

@ -29,9 +29,10 @@
#include "tcompression.h" #include "tcompression.h"
//#include "queryLog.h" //#include "queryLog.h"
#include "tudf.h" #include "tudf.h"
#include "tep.h"
#define GET_INPUT_DATA_LIST(x) ((char *)((x)->pInput)) #define GET_INPUT_DATA_LIST(x) ((char *)((x)->pInput))
#define GET_INPUT_DATA(x, y) (GET_INPUT_DATA_LIST(x) + (y) * (x)->inputBytes) #define GET_INPUT_DATA(x, y) ((char*) colDataGetData((x)->pInput, (y)))
#define GET_TS_LIST(x) ((TSKEY*)((x)->ptsList)) #define GET_TS_LIST(x) ((TSKEY*)((x)->ptsList))
#define GET_TS_DATA(x, y) (GET_TS_LIST(x)[(y)]) #define GET_TS_DATA(x, y) (GET_TS_LIST(x)[(y)])
@ -3818,7 +3819,7 @@ static void interp_function_impl(SqlFunctionCtx *pCtx) {
skey = ekey; skey = ekey;
} }
} }
assignVal(pCtx->pOutput, pCtx->pInput, pCtx->resDataInfo.bytes, pCtx->inputType); // assignVal(pCtx->pOutput, pCtx->pInput, pCtx->resDataInfo.bytes, pCtx->inputType);
} else if (type == TSDB_FILL_NEXT) { } else if (type == TSDB_FILL_NEXT) {
TSKEY ekey = skey; TSKEY ekey = skey;
char* val = NULL; char* val = NULL;
@ -4395,7 +4396,7 @@ SFunctionFpSet fpSet[1] = {
.addInput = count_function, .addInput = count_function,
.finalize = doFinalizer, .finalize = doFinalizer,
.combine = count_func_merge, .combine = count_func_merge,
} },
}; };
SAggFunctionInfo aggFunc[35] = {{ SAggFunctionInfo aggFunc[35] = {{

View File

@ -222,7 +222,7 @@ tMemBucket *tMemBucketCreate(int16_t nElemSize, int16_t dataType, double minval,
} }
pBucket->numOfSlots = DEFAULT_NUM_OF_SLOT; pBucket->numOfSlots = DEFAULT_NUM_OF_SLOT;
pBucket->bufPageSize = DEFAULT_PAGE_SIZE * 4; // 4k per page pBucket->bufPageSize = 16384 * 4; // 16k per page
pBucket->type = dataType; pBucket->type = dataType;
pBucket->bytes = nElemSize; pBucket->bytes = nElemSize;
@ -255,7 +255,7 @@ tMemBucket *tMemBucketCreate(int16_t nElemSize, int16_t dataType, double minval,
resetSlotInfo(pBucket); resetSlotInfo(pBucket);
int32_t ret = createDiskbasedBuffer(&pBucket->pBuffer, pBucket->bufPageSize, pBucket->bufPageSize * 512, 1, tsTempDir); int32_t ret = createDiskbasedBuf(&pBucket->pBuffer, pBucket->bufPageSize, pBucket->bufPageSize * 512, 1, "/tmp");
if (ret != 0) { if (ret != 0) {
tMemBucketDestroy(pBucket); tMemBucketDestroy(pBucket);
return NULL; return NULL;
@ -270,7 +270,7 @@ void tMemBucketDestroy(tMemBucket *pBucket) {
return; return;
} }
destroyResultBuf(pBucket->pBuffer); destroyDiskbasedBuf(pBucket->pBuffer);
tfree(pBucket->pSlots); tfree(pBucket->pSlots);
tfree(pBucket); tfree(pBucket);
} }
@ -348,7 +348,7 @@ int32_t tMemBucketPut(tMemBucket *pBucket, const void *data, size_t size) {
pSlot->info.data = NULL; pSlot->info.data = NULL;
} }
pSlot->info.data = getNewDataBuf(pBucket->pBuffer, groupId, &pageId); pSlot->info.data = getNewBufPage(pBucket->pBuffer, groupId, &pageId);
pSlot->info.pageId = pageId; pSlot->info.pageId = pageId;
} }

View File

@ -230,7 +230,7 @@ int32_t getExprFunctionId(SExprInfo *pExprInfo) {
} }
void assignExprInfo(SExprInfo* dst, const SExprInfo* src) { void assignExprInfo(SExprInfo* dst, const SExprInfo* src) {
assert(dst != NULL && src != NULL); assert(dst != NULL && src != NULL/* && src->base.numOfCols > 0*/);
*dst = *src; *dst = *src;
#if 0 #if 0
@ -241,8 +241,12 @@ void assignExprInfo(SExprInfo* dst, const SExprInfo* src) {
#endif #endif
dst->pExpr = exprdup(src->pExpr); dst->pExpr = exprdup(src->pExpr);
dst->base.pColumns = calloc(src->base.numOfCols, sizeof(SColumn)); if (src->base.numOfCols > 0) {
memcpy(dst->base.pColumns, src->base.pColumns, sizeof(SColumn) * src->base.numOfCols); dst->base.pColumns = calloc(src->base.numOfCols, sizeof(SColumn));
memcpy(dst->base.pColumns, src->base.pColumns, sizeof(SColumn) * src->base.numOfCols);
} else {
dst->base.pColumns = NULL;
}
memset(dst->base.param, 0, sizeof(SVariant) * tListLen(dst->base.param)); memset(dst->base.param, 0, sizeof(SVariant) * tListLen(dst->base.param));
for (int32_t j = 0; j < src->base.numOfParams; ++j) { for (int32_t j = 0; j < src->base.numOfParams; ++j) {

View File

@ -306,7 +306,7 @@ typedef struct SFilterInfo {
#define FILTER_GET_COL_FIELD_ID(fi) (((SColumnNode *)((fi)->desc))->colId) #define FILTER_GET_COL_FIELD_ID(fi) (((SColumnNode *)((fi)->desc))->colId)
#define FILTER_GET_COL_FIELD_SLOT_ID(fi) (((SColumnNode *)((fi)->desc))->slotId) #define FILTER_GET_COL_FIELD_SLOT_ID(fi) (((SColumnNode *)((fi)->desc))->slotId)
#define FILTER_GET_COL_FIELD_DESC(fi) ((SColumnNode *)((fi)->desc)) #define FILTER_GET_COL_FIELD_DESC(fi) ((SColumnNode *)((fi)->desc))
#define FILTER_GET_COL_FIELD_DATA(fi, ri) (colDataGet(((SColumnInfoData *)(fi)->data), (ri))) #define FILTER_GET_COL_FIELD_DATA(fi, ri) (colDataGetData(((SColumnInfoData *)(fi)->data), (ri)))
#define FILTER_GET_VAL_FIELD_TYPE(fi) (((SValueNode *)((fi)->desc))->node.resType.type) #define FILTER_GET_VAL_FIELD_TYPE(fi) (((SValueNode *)((fi)->desc))->node.resType.type)
#define FILTER_GET_VAL_FIELD_DATA(fi) ((char *)(fi)->data) #define FILTER_GET_VAL_FIELD_DATA(fi) ((char *)(fi)->data)
#define FILTER_GET_JSON_VAL_FIELD_DATA(fi) ((char *)(fi)->desc) #define FILTER_GET_JSON_VAL_FIELD_DATA(fi) ((char *)(fi)->desc)

View File

@ -19,6 +19,7 @@
#include "tcompare.h" #include "tcompare.h"
#include "filterInt.h" #include "filterInt.h"
#include "filter.h" #include "filter.h"
#include "tep.h"
OptrStr gOptrStr[] = { OptrStr gOptrStr[] = {
{0, "invalid"}, {0, "invalid"},
@ -2776,7 +2777,7 @@ bool filterExecuteBasedOnStatisImpl(void *pinfo, int32_t numOfRows, int8_t** p,
uint32_t unitNum = *(unitIdx++); uint32_t unitNum = *(unitIdx++);
for (uint32_t u = 0; u < unitNum; ++u) { for (uint32_t u = 0; u < unitNum; ++u) {
SFilterComUnit *cunit = &info->cunits[*(unitIdx + u)]; SFilterComUnit *cunit = &info->cunits[*(unitIdx + u)];
void *colData = colDataGet((SColumnInfoData *)cunit->colData, i); void *colData = colDataGetData((SColumnInfoData *)cunit->colData, i);
//if (FILTER_UNIT_GET_F(info, uidx)) { //if (FILTER_UNIT_GET_F(info, uidx)) {
// p[i] = FILTER_UNIT_GET_R(info, uidx); // p[i] = FILTER_UNIT_GET_R(info, uidx);
@ -2874,7 +2875,7 @@ static FORCE_INLINE bool filterExecuteImplIsNull(void *pinfo, int32_t numOfRows,
for (int32_t i = 0; i < numOfRows; ++i) { for (int32_t i = 0; i < numOfRows; ++i) {
uint32_t uidx = info->groups[0].unitIdxs[0]; uint32_t uidx = info->groups[0].unitIdxs[0];
void *colData = colDataGet((SColumnInfoData *)info->cunits[uidx].colData, i); void *colData = colDataGetData((SColumnInfoData *)info->cunits[uidx].colData, i);
if(info->cunits[uidx].dataType == TSDB_DATA_TYPE_JSON){ if(info->cunits[uidx].dataType == TSDB_DATA_TYPE_JSON){
if (!colData){ // for json->'key' is null if (!colData){ // for json->'key' is null
(*p)[i] = 1; (*p)[i] = 1;
@ -2908,7 +2909,7 @@ static FORCE_INLINE bool filterExecuteImplNotNull(void *pinfo, int32_t numOfRows
for (int32_t i = 0; i < numOfRows; ++i) { for (int32_t i = 0; i < numOfRows; ++i) {
uint32_t uidx = info->groups[0].unitIdxs[0]; uint32_t uidx = info->groups[0].unitIdxs[0];
void *colData = colDataGet((SColumnInfoData *)info->cunits[uidx].colData, i); void *colData = colDataGetData((SColumnInfoData *)info->cunits[uidx].colData, i);
if(info->cunits[uidx].dataType == TSDB_DATA_TYPE_JSON){ if(info->cunits[uidx].dataType == TSDB_DATA_TYPE_JSON){
if (!colData) { // for json->'key' is not null if (!colData) { // for json->'key' is not null
@ -2949,7 +2950,7 @@ bool filterExecuteImplRange(void *pinfo, int32_t numOfRows, int8_t** p, SColumnD
} }
for (int32_t i = 0; i < numOfRows; ++i) { for (int32_t i = 0; i < numOfRows; ++i) {
void *colData = colDataGet((SColumnInfoData *)info->cunits[0].colData, i); void *colData = colDataGetData((SColumnInfoData *)info->cunits[0].colData, i);
if (colData == NULL || isNull(colData, info->cunits[0].dataType)) { if (colData == NULL || isNull(colData, info->cunits[0].dataType)) {
all = false; all = false;
@ -2980,7 +2981,7 @@ bool filterExecuteImplMisc(void *pinfo, int32_t numOfRows, int8_t** p, SColumnDa
for (int32_t i = 0; i < numOfRows; ++i) { for (int32_t i = 0; i < numOfRows; ++i) {
uint32_t uidx = info->groups[0].unitIdxs[0]; uint32_t uidx = info->groups[0].unitIdxs[0];
void *colData = colDataGet((SColumnInfoData *)info->cunits[uidx].colData, i); void *colData = colDataGetData((SColumnInfoData *)info->cunits[uidx].colData, i);
if (colData == NULL || isNull(colData, info->cunits[uidx].dataType)) { if (colData == NULL || isNull(colData, info->cunits[uidx].dataType)) {
(*p)[i] = 0; (*p)[i] = 0;
all = false; all = false;
@ -3031,7 +3032,7 @@ bool filterExecuteImpl(void *pinfo, int32_t numOfRows, int8_t** p, SColumnDataAg
for (uint32_t u = 0; u < group->unitNum; ++u) { for (uint32_t u = 0; u < group->unitNum; ++u) {
uint32_t uidx = group->unitIdxs[u]; uint32_t uidx = group->unitIdxs[u];
SFilterComUnit *cunit = &info->cunits[uidx]; SFilterComUnit *cunit = &info->cunits[uidx];
void *colData = colDataGet((SColumnInfoData *)(cunit->colData), i); void *colData = colDataGetData((SColumnInfoData *)(cunit->colData), i);
//if (FILTER_UNIT_GET_F(info, uidx)) { //if (FILTER_UNIT_GET_F(info, uidx)) {
// p[i] = FILTER_UNIT_GET_R(info, uidx); // p[i] = FILTER_UNIT_GET_R(info, uidx);

View File

@ -226,7 +226,7 @@ void* getVectorValueAddr_default(void *src, int32_t index) {
return src; return src;
} }
void* getVectorValueAddr_VAR(void *src, int32_t index) { void* getVectorValueAddr_VAR(void *src, int32_t index) {
return colDataGet((SColumnInfoData *)src, index); return colDataGetData((SColumnInfoData *)src, index);
} }
_getValueAddr_fn_t getVectorValueAddrFn(int32_t srcType) { _getValueAddr_fn_t getVectorValueAddrFn(int32_t srcType) {

View File

@ -1,18 +1,3 @@
/*
* Copyright (c) 2019 TAOS Data, Inc. <jhtao@taosdata.com>
*
* This program is free software: you can use, redistribute, and/or modify
* it under the terms of the GNU Affero General Public License, version 3
* or later ("AGPL"), as published by the Free Software Foundation.
*
* This program is distributed in the hope that it will be useful, but WITHOUT
* ANY WARRANTY; without even the implied warranty of MERCHANTABILITY or
* FITNESS FOR A PARTICULAR PURPOSE.
*
* You should have received a copy of the GNU Affero General Public License
* along with this program. If not, see <http://www.gnu.org/licenses/>.
*/
#define _DEFAULT_SOURCE #define _DEFAULT_SOURCE
#include "tpagedbuf.h" #include "tpagedbuf.h"
#include "taoserror.h" #include "taoserror.h"
@ -23,27 +8,22 @@
#define GET_DATA_PAYLOAD(_p) ((char*)(_p)->pData + POINTER_BYTES) #define GET_DATA_PAYLOAD(_p) ((char*)(_p)->pData + POINTER_BYTES)
#define NO_IN_MEM_AVAILABLE_PAGES(_b) (listNEles((_b)->lruList) >= (_b)->inMemPages) #define NO_IN_MEM_AVAILABLE_PAGES(_b) (listNEles((_b)->lruList) >= (_b)->inMemPages)
typedef struct SFreeListItem {
int32_t offset;
int32_t len;
} SFreeListItem;
typedef struct SPageDiskInfo { typedef struct SPageDiskInfo {
int64_t offset; int64_t offset;
int32_t length; int32_t length;
} SPageDiskInfo; } SPageDiskInfo, SFreeListItem;
typedef struct SPageInfo { struct SPageInfo {
SListNode* pn; // point to list node SListNode* pn; // point to list node struct
void* pData; void* pData;
int64_t offset; int64_t offset;
int32_t pageId; int32_t pageId;
int32_t length : 30; int32_t length : 29;
bool used : 1; // set current page is in used bool used : 1; // set current page is in used
bool dirty : 1; // set current buffer page is dirty or not bool dirty : 1; // set current buffer page is dirty or not
} SPageInfo; };
typedef struct SDiskbasedBuf { struct SDiskbasedBuf {
int32_t numOfPages; int32_t numOfPages;
int64_t totalBufSize; int64_t totalBufSize;
uint64_t fileSize; // disk file size uint64_t fileSize; // disk file size
@ -52,7 +32,8 @@ typedef struct SDiskbasedBuf {
char* path; // file path char* path; // file path
int32_t pageSize; // current used page size int32_t pageSize; // current used page size
int32_t inMemPages; // numOfPages that are allocated in memory int32_t inMemPages; // numOfPages that are allocated in memory
SHashObj* groupSet; // id hash table SList* freePgList; // free page list
SHashObj* groupSet; // id hash table, todo remove it
SHashObj* all; SHashObj* all;
SList* lruList; SList* lruList;
void* emptyDummyIdList; // dummy id list void* emptyDummyIdList; // dummy id list
@ -64,51 +45,7 @@ typedef struct SDiskbasedBuf {
uint64_t qId; // for debug purpose uint64_t qId; // for debug purpose
bool printStatis; // Print statistics info when closing this buffer. bool printStatis; // Print statistics info when closing this buffer.
SDiskbasedBufStatis statis; SDiskbasedBufStatis statis;
} SDiskbasedBuf; };
static void printStatisData(const SDiskbasedBuf* pBuf);
int32_t createDiskbasedBuffer(SDiskbasedBuf** pBuf, int32_t pagesize, int32_t inMemBufSize, uint64_t qId,
const char* dir) {
*pBuf = calloc(1, sizeof(SDiskbasedBuf));
SDiskbasedBuf* pResBuf = *pBuf;
if (pResBuf == NULL) {
return TSDB_CODE_OUT_OF_MEMORY;
}
pResBuf->pageSize = pagesize;
pResBuf->numOfPages = 0; // all pages are in buffer in the first place
pResBuf->totalBufSize = 0;
pResBuf->inMemPages = inMemBufSize / pagesize; // maximum allowed pages, it is a soft limit.
pResBuf->allocateId = -1;
pResBuf->comp = true;
pResBuf->pFile = NULL;
pResBuf->qId = qId;
pResBuf->fileSize = 0;
// at least more than 2 pages must be in memory
assert(inMemBufSize >= pagesize * 2);
pResBuf->lruList = tdListNew(POINTER_BYTES);
// init id hash table
pResBuf->groupSet = taosHashInit(10, taosGetDefaultHashFunction(TSDB_DATA_TYPE_INT), true, false);
pResBuf->assistBuf = malloc(pResBuf->pageSize + 2); // EXTRA BYTES
pResBuf->all = taosHashInit(10, taosGetDefaultHashFunction(TSDB_DATA_TYPE_INT), true, false);
char path[PATH_MAX] = {0};
taosGetTmpfilePath(dir, "qbuf", path);
pResBuf->path = strdup(path);
pResBuf->emptyDummyIdList = taosArrayInit(1, sizeof(int32_t));
// qDebug("QInfo:0x%"PRIx64" create resBuf for output, page size:%d, inmem buf pages:%d, file:%s", qId,
// pResBuf->pageSize,
// pResBuf->inMemPages, pResBuf->path);
return TSDB_CODE_SUCCESS;
}
static int32_t createDiskFile(SDiskbasedBuf* pBuf) { static int32_t createDiskFile(SDiskbasedBuf* pBuf) {
// pBuf->file = fopen(pBuf->path, "wb+"); // pBuf->file = fopen(pBuf->path, "wb+");
@ -139,8 +76,7 @@ static char* doDecompressData(void* data, int32_t srcSize, int32_t* dst, SDiskba
return data; return data;
} }
*dst = tsDecompressString(data, srcSize, 1, pBuf->assistBuf, pBuf->pageSize + sizeof(SFilePage), ONE_STAGE_COMP, NULL, *dst = tsDecompressString(data, srcSize, 1, pBuf->assistBuf, pBuf->pageSize, ONE_STAGE_COMP, NULL, 0);
0);
if (*dst > 0) { if (*dst > 0) {
memcpy(data, pBuf->assistBuf, *dst); memcpy(data, pBuf->assistBuf, *dst);
} }
@ -156,10 +92,10 @@ static uint64_t allocatePositionInFile(SDiskbasedBuf* pBuf, size_t size) {
size_t num = taosArrayGetSize(pBuf->pFree); size_t num = taosArrayGetSize(pBuf->pFree);
for (int32_t i = 0; i < num; ++i) { for (int32_t i = 0; i < num; ++i) {
SFreeListItem* pi = taosArrayGet(pBuf->pFree, i); SFreeListItem* pi = taosArrayGet(pBuf->pFree, i);
if (pi->len >= size) { if (pi->length >= size) {
offset = pi->offset; offset = pi->offset;
pi->offset += (int32_t)size; pi->offset += (int32_t)size;
pi->len -= (int32_t)size; pi->length -= (int32_t)size;
return offset; return offset;
} }
@ -170,81 +106,100 @@ static uint64_t allocatePositionInFile(SDiskbasedBuf* pBuf, size_t size) {
} }
} }
static void setPageNotInBuf(SPageInfo* pPageInfo) { pPageInfo->pData = NULL; }
static FORCE_INLINE size_t getAllocPageSize(int32_t pageSize) { return pageSize + POINTER_BYTES + 2; }
/**
* +--------------------------+-------------------+--------------+
* | PTR to SPageInfo (8bytes)| Payload (PageSize)| 2 Extra Bytes|
* +--------------------------+-------------------+--------------+
* @param pBuf
* @param pg
* @return
*/
static char* doFlushPageToDisk(SDiskbasedBuf* pBuf, SPageInfo* pg) { static char* doFlushPageToDisk(SDiskbasedBuf* pBuf, SPageInfo* pg) {
assert(!pg->used && pg->pData != NULL); assert(!pg->used && pg->pData != NULL);
int32_t size = -1; int32_t size = pBuf->pageSize;
char* t = NULL; char* t = NULL;
if (pg->offset == -1 || pg->dirty) { if (pg->offset == -1 || pg->dirty) {
SFilePage* pPage = (SFilePage*)GET_DATA_PAYLOAD(pg); void* payload = GET_DATA_PAYLOAD(pg);
t = doCompressData(pPage->data, pBuf->pageSize, &size, pBuf); t = doCompressData(payload, pBuf->pageSize, &size, pBuf);
assert(size >= 0);
} }
// this page is flushed to disk for the first time // this page is flushed to disk for the first time
if (pg->offset == -1) { if (pg->dirty) {
assert(pg->dirty == true); if (pg->offset == -1) {
assert(pg->dirty == true);
pg->offset = allocatePositionInFile(pBuf, size);
pBuf->nextPos += size;
int32_t ret = taosLSeekFile(pBuf->pFile, pg->offset, SEEK_SET);
if (ret != 0) {
terrno = TAOS_SYSTEM_ERROR(errno);
return NULL;
}
ret = (int32_t)taosWriteFile(pBuf->pFile, t, size);
if (ret != size) {
terrno = TAOS_SYSTEM_ERROR(errno);
return NULL;
}
if (pBuf->fileSize < pg->offset + size) {
pBuf->fileSize = pg->offset + size;
}
pBuf->statis.flushBytes += size;
pBuf->statis.flushPages += 1;
} else if (pg->dirty) {
// length becomes greater, current space is not enough, allocate new place, otherwise, do nothing
if (pg->length < size) {
// 1. add current space to free list
SPageDiskInfo dinfo = {.length = pg->length, .offset = pg->offset};
taosArrayPush(pBuf->pFree, &dinfo);
// 2. allocate new position, and update the info
pg->offset = allocatePositionInFile(pBuf, size); pg->offset = allocatePositionInFile(pBuf, size);
pBuf->nextPos += size; pBuf->nextPos += size;
}
// 3. write to disk. int32_t ret = taosLSeekFile(pBuf->pFile, pg->offset, SEEK_SET);
int32_t ret = taosLSeekFile(pBuf->pFile, pg->offset, SEEK_SET); if (ret != 0) {
if (ret != 0) { terrno = TAOS_SYSTEM_ERROR(errno);
terrno = TAOS_SYSTEM_ERROR(errno); return NULL;
return NULL; }
}
ret = (int32_t)taosWriteFile(pBuf->pFile, t, size); ret = (int32_t)taosWriteFile(pBuf->pFile, t, size);
if (ret != size) { if (ret != size) {
terrno = TAOS_SYSTEM_ERROR(errno); terrno = TAOS_SYSTEM_ERROR(errno);
return NULL; return NULL;
} }
if (pBuf->fileSize < pg->offset + size) { if (pBuf->fileSize < pg->offset + size) {
pBuf->fileSize = pg->offset + size; pBuf->fileSize = pg->offset + size;
} }
pBuf->statis.flushBytes += size; pBuf->statis.flushBytes += size;
pBuf->statis.flushPages += 1; pBuf->statis.flushPages += 1;
} else {
// length becomes greater, current space is not enough, allocate new place, otherwise, do nothing
if (pg->length < size) {
// 1. add current space to free list
SPageDiskInfo dinfo = {.length = pg->length, .offset = pg->offset};
taosArrayPush(pBuf->pFree, &dinfo);
// 2. allocate new position, and update the info
pg->offset = allocatePositionInFile(pBuf, size);
pBuf->nextPos += size;
}
// 3. write to disk.
int32_t ret = taosLSeekFile(pBuf->pFile, pg->offset, SEEK_SET);
if (ret != 0) {
terrno = TAOS_SYSTEM_ERROR(errno);
return NULL;
}
ret = (int32_t)taosWriteFile(pBuf->pFile, t, size);
if (ret != size) {
terrno = TAOS_SYSTEM_ERROR(errno);
return NULL;
}
if (pBuf->fileSize < pg->offset + size) {
pBuf->fileSize = pg->offset + size;
}
pBuf->statis.flushBytes += size;
pBuf->statis.flushPages += 1;
}
} else { // NOTE: the size may be -1, the this recycle page has not been flushed to disk yet.
size = pg->length;
if (size == -1) {
printf("----\n");
}
} }
ASSERT(size > 0 || (pg->offset == -1 && pg->length == -1));
char* pDataBuf = pg->pData; char* pDataBuf = pg->pData;
memset(pDataBuf, 0, pBuf->pageSize + sizeof(SFilePage)); memset(pDataBuf, 0, getAllocPageSize(pBuf->pageSize));
pg->pData = NULL; // this means the data is not in buffer
pg->length = size;
pg->dirty = false;
pg->length = size; // on disk size
return pDataBuf; return pDataBuf;
} }
@ -259,7 +214,11 @@ static char* flushPageToDisk(SDiskbasedBuf* pBuf, SPageInfo* pg) {
} }
} }
return doFlushPageToDisk(pBuf, pg); char* p = doFlushPageToDisk(pBuf, pg);
setPageNotInBuf(pg);
pg->dirty = false;
return p;
} }
// load file block data in disk // load file block data in disk
@ -270,8 +229,8 @@ static int32_t loadPageFromDisk(SDiskbasedBuf* pBuf, SPageInfo* pg) {
return ret; return ret;
} }
SFilePage* pPage = (SFilePage*)GET_DATA_PAYLOAD(pg); void* pPage = (void*)GET_DATA_PAYLOAD(pg);
ret = (int32_t)taosReadFile(pBuf->pFile, pPage->data, pg->length); ret = (int32_t)taosReadFile(pBuf->pFile, pPage, pg->length);
if (ret != pg->length) { if (ret != pg->length) {
ret = TAOS_SYSTEM_ERROR(errno); ret = TAOS_SYSTEM_ERROR(errno);
return ret; return ret;
@ -281,7 +240,7 @@ static int32_t loadPageFromDisk(SDiskbasedBuf* pBuf, SPageInfo* pg) {
pBuf->statis.loadPages += 1; pBuf->statis.loadPages += 1;
int32_t fullSize = 0; int32_t fullSize = 0;
doDecompressData(pPage->data, pg->length, &fullSize, pBuf); doDecompressData(pPage, pg->length, &fullSize, pBuf);
return 0; return 0;
} }
@ -307,7 +266,7 @@ static SPageInfo* registerPage(SDiskbasedBuf* pBuf, int32_t groupId, int32_t pag
pBuf->numOfPages += 1; pBuf->numOfPages += 1;
SPageInfo* ppi = malloc(sizeof(SPageInfo)); //{ .info = PAGE_INFO_INITIALIZER, .pageId = pageId, .pn = NULL}; SPageInfo* ppi = malloc(sizeof(SPageInfo));
ppi->pageId = pageId; ppi->pageId = pageId;
ppi->pData = NULL; ppi->pData = NULL;
@ -325,16 +284,27 @@ static SListNode* getEldestUnrefedPage(SDiskbasedBuf* pBuf) {
SListNode* pn = NULL; SListNode* pn = NULL;
while ((pn = tdListNext(&iter)) != NULL) { while ((pn = tdListNext(&iter)) != NULL) {
assert(pn != NULL);
SPageInfo* pageInfo = *(SPageInfo**)pn->data; SPageInfo* pageInfo = *(SPageInfo**)pn->data;
assert(pageInfo->pageId >= 0 && pageInfo->pn == pn); assert(pageInfo->pageId >= 0 && pageInfo->pn == pn);
if (!pageInfo->used) { if (!pageInfo->used) {
// printf("%d is chosen\n", pageInfo->pageId);
break; break;
} else {
// printf("page %d is used, dirty:%d\n", pageInfo->pageId, pageInfo->dirty);
} }
} }
// int32_t pos = listNEles(pBuf->lruList);
// SListIter iter1 = {0};
// tdListInitIter(pBuf->lruList, &iter1, TD_LIST_BACKWARD);
// SListNode* pn1 = NULL;
// while((pn1 = tdListNext(&iter1)) != NULL) {
// SPageInfo* pageInfo = *(SPageInfo**) pn1->data;
// printf("page %d is used, dirty:%d, pos:%d\n", pageInfo->pageId, pageInfo->dirty, pos - 1);
// pos -= 1;
// }
return pn; return pn;
} }
@ -378,11 +348,60 @@ static void lruListMoveToFront(SList* pList, SPageInfo* pi) {
tdListPrependNode(pList, pi->pn); tdListPrependNode(pList, pi->pn);
} }
static FORCE_INLINE size_t getAllocPageSize(int32_t pageSize) { static SPageInfo* getPageInfoFromPayload(void* page) {
return pageSize + POINTER_BYTES + 2 + sizeof(SFilePage); int32_t offset = offsetof(SPageInfo, pData);
char* p = page - offset;
SPageInfo* ppi = ((SPageInfo**)p)[0];
return ppi;
} }
SFilePage* getNewDataBuf(SDiskbasedBuf* pBuf, int32_t groupId, int32_t* pageId) { int32_t createDiskbasedBuf(SDiskbasedBuf** pBuf, int32_t pagesize, int32_t inMemBufSize, uint64_t qId,
const char* dir) {
*pBuf = calloc(1, sizeof(SDiskbasedBuf));
SDiskbasedBuf* pPBuf = *pBuf;
if (pPBuf == NULL) {
return TSDB_CODE_OUT_OF_MEMORY;
}
pPBuf->pageSize = pagesize;
pPBuf->numOfPages = 0; // all pages are in buffer in the first place
pPBuf->totalBufSize = 0;
pPBuf->inMemPages = inMemBufSize / pagesize; // maximum allowed pages, it is a soft limit.
pPBuf->allocateId = -1;
pPBuf->comp = true;
pPBuf->pFile = NULL;
pPBuf->qId = qId;
pPBuf->fileSize = 0;
pPBuf->pFree = taosArrayInit(4, sizeof(SFreeListItem));
pPBuf->freePgList = tdListNew(POINTER_BYTES);
// at least more than 2 pages must be in memory
assert(inMemBufSize >= pagesize * 2);
pPBuf->lruList = tdListNew(POINTER_BYTES);
// init id hash table
_hash_fn_t fn = taosGetDefaultHashFunction(TSDB_DATA_TYPE_INT);
pPBuf->groupSet = taosHashInit(10, fn, true, false);
pPBuf->assistBuf = malloc(pPBuf->pageSize + 2); // EXTRA BYTES
pPBuf->all = taosHashInit(10, fn, true, false);
char path[PATH_MAX] = {0};
taosGetTmpfilePath(dir, "paged-buf", path);
pPBuf->path = strdup(path);
pPBuf->emptyDummyIdList = taosArrayInit(1, sizeof(int32_t));
// qDebug("QInfo:0x%"PRIx64" create resBuf for output, page size:%d, inmem buf pages:%d, file:%s", qId,
// pPBuf->pageSize,
// pPBuf->inMemPages, pPBuf->path);
return TSDB_CODE_SUCCESS;
}
void* getNewBufPage(SDiskbasedBuf* pBuf, int32_t groupId, int32_t* pageId) {
pBuf->statis.getPages += 1; pBuf->statis.getPages += 1;
char* availablePage = NULL; char* availablePage = NULL;
@ -391,23 +410,34 @@ SFilePage* getNewDataBuf(SDiskbasedBuf* pBuf, int32_t groupId, int32_t* pageId)
// Failed to allocate a new buffer page, and there is an error occurs. // Failed to allocate a new buffer page, and there is an error occurs.
if (availablePage == NULL) { if (availablePage == NULL) {
assert(0);
return NULL; return NULL;
} }
} }
// register new id in this group SPageInfo* pi = NULL;
*pageId = (++pBuf->allocateId); if (listNEles(pBuf->freePgList) != 0) {
SListNode* pItem = tdListPopHead(pBuf->freePgList);
pi = *(SPageInfo**)pItem->data;
pi->used = true;
*pageId = pi->pageId;
tfree(pItem);
} else { // create a new pageinfo
// register new id in this group
*pageId = (++pBuf->allocateId);
// register page id info // register page id info
SPageInfo* pi = registerPage(pBuf, groupId, *pageId); pi = registerPage(pBuf, groupId, *pageId);
// add to hash map
taosHashPut(pBuf->all, pageId, sizeof(int32_t), &pi, POINTER_BYTES);
pBuf->totalBufSize += pBuf->pageSize;
}
// add to LRU list // add to LRU list
assert(listNEles(pBuf->lruList) < pBuf->inMemPages && pBuf->inMemPages > 0); assert(listNEles(pBuf->lruList) < pBuf->inMemPages && pBuf->inMemPages > 0);
lruListPushFront(pBuf->lruList, pi); lruListPushFront(pBuf->lruList, pi);
// add to hash map
taosHashPut(pBuf->all, pageId, sizeof(int32_t), &pi, POINTER_BYTES);
// allocate buf // allocate buf
if (availablePage == NULL) { if (availablePage == NULL) {
pi->pData = calloc(1, getAllocPageSize(pBuf->pageSize)); // add extract bytes in case of zipped buffer increased. pi->pData = calloc(1, getAllocPageSize(pBuf->pageSize)); // add extract bytes in case of zipped buffer increased.
@ -415,15 +445,11 @@ SFilePage* getNewDataBuf(SDiskbasedBuf* pBuf, int32_t groupId, int32_t* pageId)
pi->pData = availablePage; pi->pData = availablePage;
} }
pBuf->totalBufSize += pBuf->pageSize;
((void**)pi->pData)[0] = pi; ((void**)pi->pData)[0] = pi;
pi->used = true;
return (void*)(GET_DATA_PAYLOAD(pi)); return (void*)(GET_DATA_PAYLOAD(pi));
} }
SFilePage* getBufPage(SDiskbasedBuf* pBuf, int32_t id) { void* getBufPage(SDiskbasedBuf* pBuf, int32_t id) {
assert(pBuf != NULL && id >= 0); assert(pBuf != NULL && id >= 0);
pBuf->statis.getPages += 1; pBuf->statis.getPages += 1;
@ -444,7 +470,6 @@ SFilePage* getBufPage(SDiskbasedBuf* pBuf, int32_t id) {
(*pi)->used = true; (*pi)->used = true;
return (void*)(GET_DATA_PAYLOAD(*pi)); return (void*)(GET_DATA_PAYLOAD(*pi));
} else { // not in memory } else { // not in memory
assert((*pi)->pData == NULL && (*pi)->pn == NULL && (*pi)->length >= 0 && (*pi)->offset >= 0); assert((*pi)->pData == NULL && (*pi)->pn == NULL && (*pi)->length >= 0 && (*pi)->offset >= 0);
@ -462,6 +487,7 @@ SFilePage* getBufPage(SDiskbasedBuf* pBuf, int32_t id) {
(*pi)->pData = availablePage; (*pi)->pData = availablePage;
} }
// set the ptr to the new SPageInfo
((void**)((*pi)->pData))[0] = (*pi); ((void**)((*pi)->pData))[0] = (*pi);
lruListPushFront(pBuf->lruList, *pi); lruListPushFront(pBuf->lruList, *pi);
@ -478,21 +504,18 @@ SFilePage* getBufPage(SDiskbasedBuf* pBuf, int32_t id) {
void releaseBufPage(SDiskbasedBuf* pBuf, void* page) { void releaseBufPage(SDiskbasedBuf* pBuf, void* page) {
assert(pBuf != NULL && page != NULL); assert(pBuf != NULL && page != NULL);
int32_t offset = offsetof(SPageInfo, pData); SPageInfo* ppi = getPageInfoFromPayload(page);
char* p = page - offset;
SPageInfo* ppi = ((SPageInfo**)p)[0];
releaseBufPageInfo(pBuf, ppi); releaseBufPageInfo(pBuf, ppi);
} }
void releaseBufPageInfo(SDiskbasedBuf* pBuf, SPageInfo* pi) { void releaseBufPageInfo(SDiskbasedBuf* pBuf, SPageInfo* pi) {
assert(pi->pData != NULL && pi->used); assert(pi->pData != NULL && pi->used == true);
pi->used = false; pi->used = false;
pBuf->statis.releasePages += 1; pBuf->statis.releasePages += 1;
} }
size_t getNumOfResultBufGroupId(const SDiskbasedBuf* pBuf) { return taosHashGetSize(pBuf->groupSet); } size_t getNumOfBufGroupId(const SDiskbasedBuf* pBuf) { return taosHashGetSize(pBuf->groupSet); }
size_t getTotalBufSize(const SDiskbasedBuf* pBuf) { return (size_t)pBuf->totalBufSize; } size_t getTotalBufSize(const SDiskbasedBuf* pBuf) { return (size_t)pBuf->totalBufSize; }
@ -507,12 +530,12 @@ SIDList getDataBufPagesIdList(SDiskbasedBuf* pBuf, int32_t groupId) {
} }
} }
void destroyResultBuf(SDiskbasedBuf* pBuf) { void destroyDiskbasedBuf(SDiskbasedBuf* pBuf) {
if (pBuf == NULL) { if (pBuf == NULL) {
return; return;
} }
printStatisData(pBuf); dBufPrintStatis(pBuf);
if (pBuf->pFile != NULL) { if (pBuf->pFile != NULL) {
uDebug( uDebug(
@ -553,7 +576,11 @@ void destroyResultBuf(SDiskbasedBuf* pBuf) {
} }
tdListFree(pBuf->lruList); tdListFree(pBuf->lruList);
tdListFree(pBuf->freePgList);
taosArrayDestroy(pBuf->emptyDummyIdList); taosArrayDestroy(pBuf->emptyDummyIdList);
taosArrayDestroy(pBuf->pFree);
taosHashCleanup(pBuf->groupSet); taosHashCleanup(pBuf->groupSet);
taosHashCleanup(pBuf->all); taosHashCleanup(pBuf->all);
@ -578,19 +605,32 @@ int32_t getNumOfInMemBufPages(const SDiskbasedBuf* pBuf) { return pBuf->inMemPag
bool isAllDataInMemBuf(const SDiskbasedBuf* pBuf) { return pBuf->fileSize == 0; } bool isAllDataInMemBuf(const SDiskbasedBuf* pBuf) { return pBuf->fileSize == 0; }
void setBufPageDirty(SFilePage* pPage, bool dirty) { void setBufPageDirty(void* pPage, bool dirty) {
int32_t offset = offsetof(SPageInfo, pData); // todo extract method SPageInfo* ppi = getPageInfoFromPayload(pPage);
char* p = (char*)pPage - offset;
SPageInfo* ppi = ((SPageInfo**)p)[0];
ppi->dirty = dirty; ppi->dirty = dirty;
} }
void printStatisBeforeClose(SDiskbasedBuf* pBuf) { pBuf->printStatis = true; } void setBufPageCompressOnDisk(SDiskbasedBuf* pBuf, bool comp) { pBuf->comp = comp; }
void dBufSetBufPageRecycled(SDiskbasedBuf* pBuf, void* pPage) {
SPageInfo* ppi = getPageInfoFromPayload(pPage);
ppi->used = false;
ppi->dirty = false;
// add this pageinfo into the free page info list
SListNode* pNode = tdListPopNode(pBuf->lruList, ppi->pn);
tfree(ppi->pData);
tfree(pNode);
tdListAppend(pBuf->freePgList, &ppi);
}
void dBufSetPrintInfo(SDiskbasedBuf* pBuf) { pBuf->printStatis = true; }
SDiskbasedBufStatis getDBufStatis(const SDiskbasedBuf* pBuf) { return pBuf->statis; } SDiskbasedBufStatis getDBufStatis(const SDiskbasedBuf* pBuf) { return pBuf->statis; }
void printStatisData(const SDiskbasedBuf* pBuf) { void dBufPrintStatis(const SDiskbasedBuf* pBuf) {
if (!pBuf->printStatis) { if (!pBuf->printStatis) {
return; return;
} }

View File

@ -13,144 +13,144 @@ namespace {
// simple test // simple test
void simpleTest() { void simpleTest() {
SDiskbasedBuf* pResultBuf = NULL; SDiskbasedBuf* pResultBuf = NULL;
int32_t ret = createDiskbasedBuffer(&pResultBuf, 1024, 4096, 1, "/tmp/"); int32_t ret = createDiskbasedBuf(&pResultBuf, 1024, 4096, 1, "/tmp/");
int32_t pageId = 0; int32_t pageId = 0;
int32_t groupId = 0; int32_t groupId = 0;
SFilePage* pBufPage = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
ASSERT_TRUE(pBufPage != NULL); ASSERT_TRUE(pBufPage != NULL);
ASSERT_EQ(getTotalBufSize(pResultBuf), 1024); ASSERT_EQ(getTotalBufSize(pResultBuf), 1024);
SIDList list = getDataBufPagesIdList(pResultBuf, groupId); SIDList list = getDataBufPagesIdList(pResultBuf, groupId);
ASSERT_EQ(taosArrayGetSize(list), 1); ASSERT_EQ(taosArrayGetSize(list), 1);
ASSERT_EQ(getNumOfResultBufGroupId(pResultBuf), 1); ASSERT_EQ(getNumOfBufGroupId(pResultBuf), 1);
releaseBufPage(pResultBuf, pBufPage); releaseBufPage(pResultBuf, pBufPage);
SFilePage* pBufPage1 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage1 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t = getBufPage(pResultBuf, pageId); SFilePage* t = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t == pBufPage1); ASSERT_TRUE(t == pBufPage1);
SFilePage* pBufPage2 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage2 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t1 = getBufPage(pResultBuf, pageId); SFilePage* t1 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t1 == pBufPage2); ASSERT_TRUE(t1 == pBufPage2);
SFilePage* pBufPage3 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage3 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t2 = getBufPage(pResultBuf, pageId); SFilePage* t2 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t2 == pBufPage3); ASSERT_TRUE(t2 == pBufPage3);
SFilePage* pBufPage4 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage4 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t3 = getBufPage(pResultBuf, pageId); SFilePage* t3 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t3 == pBufPage4); ASSERT_TRUE(t3 == pBufPage4);
SFilePage* pBufPage5 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage5 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t4 = getBufPage(pResultBuf, pageId); SFilePage* t4 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t4 == pBufPage5); ASSERT_TRUE(t4 == pBufPage5);
destroyResultBuf(pResultBuf); destroyDiskbasedBuf(pResultBuf);
} }
void writeDownTest() { void writeDownTest() {
SDiskbasedBuf* pResultBuf = NULL; SDiskbasedBuf* pResultBuf = NULL;
int32_t ret = createDiskbasedBuffer(&pResultBuf, 1024, 4*1024, 1, "/tmp/"); int32_t ret = createDiskbasedBuf(&pResultBuf, 1024, 4*1024, 1, "/tmp/");
int32_t pageId = 0; int32_t pageId = 0;
int32_t writePageId = 0; int32_t writePageId = 0;
int32_t groupId = 0; int32_t groupId = 0;
int32_t nx = 12345; int32_t nx = 12345;
SFilePage* pBufPage = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
ASSERT_TRUE(pBufPage != NULL); ASSERT_TRUE(pBufPage != NULL);
*(int32_t*)(pBufPage->data) = nx; *(int32_t*)(pBufPage->data) = nx;
writePageId = pageId; writePageId = pageId;
releaseBufPage(pResultBuf, pBufPage); releaseBufPage(pResultBuf, pBufPage);
SFilePage* pBufPage1 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage1 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t1 = getBufPage(pResultBuf, pageId); SFilePage* t1 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t1 == pBufPage1); ASSERT_TRUE(t1 == pBufPage1);
ASSERT_TRUE(pageId == 1); ASSERT_TRUE(pageId == 1);
SFilePage* pBufPage2 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage2 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t2 = getBufPage(pResultBuf, pageId); SFilePage* t2 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t2 == pBufPage2); ASSERT_TRUE(t2 == pBufPage2);
ASSERT_TRUE(pageId == 2); ASSERT_TRUE(pageId == 2);
SFilePage* pBufPage3 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage3 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t3 = getBufPage(pResultBuf, pageId); SFilePage* t3 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t3 == pBufPage3); ASSERT_TRUE(t3 == pBufPage3);
ASSERT_TRUE(pageId == 3); ASSERT_TRUE(pageId == 3);
SFilePage* pBufPage4 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage4 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t4 = getBufPage(pResultBuf, pageId); SFilePage* t4 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t4 == pBufPage4); ASSERT_TRUE(t4 == pBufPage4);
ASSERT_TRUE(pageId == 4); ASSERT_TRUE(pageId == 4);
releaseBufPage(pResultBuf, t4); releaseBufPage(pResultBuf, t4);
// flush the written page to disk, and read it out again // flush the written page to disk, and read it out again
SFilePage* pBufPagex = getBufPage(pResultBuf, writePageId); SFilePage* pBufPagex = static_cast<SFilePage*>(getBufPage(pResultBuf, writePageId));
ASSERT_EQ(*(int32_t*)pBufPagex->data, nx); ASSERT_EQ(*(int32_t*)pBufPagex->data, nx);
SArray* pa = getDataBufPagesIdList(pResultBuf, groupId); SArray* pa = getDataBufPagesIdList(pResultBuf, groupId);
ASSERT_EQ(taosArrayGetSize(pa), 5); ASSERT_EQ(taosArrayGetSize(pa), 5);
destroyResultBuf(pResultBuf); destroyDiskbasedBuf(pResultBuf);
} }
void recyclePageTest() { void recyclePageTest() {
SDiskbasedBuf* pResultBuf = NULL; SDiskbasedBuf* pResultBuf = NULL;
int32_t ret = createDiskbasedBuffer(&pResultBuf, 1024, 4*1024, 1, "/tmp/"); int32_t ret = createDiskbasedBuf(&pResultBuf, 1024, 4*1024, 1, "/tmp/");
int32_t pageId = 0; int32_t pageId = 0;
int32_t writePageId = 0; int32_t writePageId = 0;
int32_t groupId = 0; int32_t groupId = 0;
int32_t nx = 12345; int32_t nx = 12345;
SFilePage* pBufPage = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
ASSERT_TRUE(pBufPage != NULL); ASSERT_TRUE(pBufPage != NULL);
releaseBufPage(pResultBuf, pBufPage); releaseBufPage(pResultBuf, pBufPage);
SFilePage* pBufPage1 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage1 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t1 = getBufPage(pResultBuf, pageId); SFilePage* t1 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t1 == pBufPage1); ASSERT_TRUE(t1 == pBufPage1);
ASSERT_TRUE(pageId == 1); ASSERT_TRUE(pageId == 1);
SFilePage* pBufPage2 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage2 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t2 = getBufPage(pResultBuf, pageId); SFilePage* t2 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t2 == pBufPage2); ASSERT_TRUE(t2 == pBufPage2);
ASSERT_TRUE(pageId == 2); ASSERT_TRUE(pageId == 2);
SFilePage* pBufPage3 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage3 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t3 = getBufPage(pResultBuf, pageId); SFilePage* t3 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t3 == pBufPage3); ASSERT_TRUE(t3 == pBufPage3);
ASSERT_TRUE(pageId == 3); ASSERT_TRUE(pageId == 3);
SFilePage* pBufPage4 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage4 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t4 = getBufPage(pResultBuf, pageId); SFilePage* t4 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t4 == pBufPage4); ASSERT_TRUE(t4 == pBufPage4);
ASSERT_TRUE(pageId == 4); ASSERT_TRUE(pageId == 4);
releaseBufPage(pResultBuf, t4); releaseBufPage(pResultBuf, t4);
SFilePage* pBufPage5 = getNewDataBuf(pResultBuf, groupId, &pageId); SFilePage* pBufPage5 = static_cast<SFilePage*>(getNewBufPage(pResultBuf, groupId, &pageId));
SFilePage* t5 = getBufPage(pResultBuf, pageId); SFilePage* t5 = static_cast<SFilePage*>(getBufPage(pResultBuf, pageId));
ASSERT_TRUE(t5 == pBufPage5); ASSERT_TRUE(t5 == pBufPage5);
ASSERT_TRUE(pageId == 5); ASSERT_TRUE(pageId == 5);
// flush the written page to disk, and read it out again // flush the written page to disk, and read it out again
SFilePage* pBufPagex = getBufPage(pResultBuf, writePageId); SFilePage* pBufPagex = static_cast<SFilePage*>(getBufPage(pResultBuf, writePageId));
*(int32_t*)(pBufPagex->data) = nx; *(int32_t*)(pBufPagex->data) = nx;
writePageId = pageId; // update the data writePageId = pageId; // update the data
releaseBufPage(pResultBuf, pBufPagex); releaseBufPage(pResultBuf, pBufPagex);
SFilePage* pBufPagex1 = getBufPage(pResultBuf, 1); SFilePage* pBufPagex1 = static_cast<SFilePage*>(getBufPage(pResultBuf, 1));
SArray* pa = getDataBufPagesIdList(pResultBuf, groupId); SArray* pa = getDataBufPagesIdList(pResultBuf, groupId);
ASSERT_EQ(taosArrayGetSize(pa), 6); ASSERT_EQ(taosArrayGetSize(pa), 6);
destroyResultBuf(pResultBuf); destroyDiskbasedBuf(pResultBuf);
} }
} // namespace } // namespace